Skip to content

Commit 4a3e60d

Browse files
authored
[FLINK-16686][State] Set classloader for compaction filters (#27073)
1 parent 714d1ba commit 4a3e60d

File tree

3 files changed

+149
-9
lines changed

3 files changed

+149
-9
lines changed
Lines changed: 93 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -0,0 +1,93 @@
1+
/*
2+
* Licensed to the Apache Software Foundation (ASF) under one
3+
* or more contributor license agreements. See the NOTICE file
4+
* distributed with this work for additional information
5+
* regarding copyright ownership. The ASF licenses this file
6+
* to you under the Apache License, Version 2.0 (the
7+
* "License"); you may not use this file except in compliance
8+
* with the License. You may obtain a copy of the License at
9+
*
10+
* http://www.apache.org/licenses/LICENSE-2.0
11+
*
12+
* Unless required by applicable law or agreed to in writing, software
13+
* distributed under the License is distributed on an "AS IS" BASIS,
14+
* WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
15+
* See the License for the specific language governing permissions and
16+
* limitations under the License.
17+
*/
18+
19+
package org.apache.flink.runtime.state.ttl;
20+
21+
import org.apache.flink.api.common.ExecutionConfig;
22+
import org.apache.flink.api.common.serialization.SerializerConfig;
23+
import org.apache.flink.api.common.serialization.SerializerConfigImpl;
24+
import org.apache.flink.api.java.typeutils.runtime.kryo.KryoSerializer;
25+
import org.apache.flink.configuration.Configuration;
26+
import org.apache.flink.configuration.PipelineOptions;
27+
28+
import java.util.Arrays;
29+
import java.util.Collections;
30+
31+
/** Test suite for {@link TtlListState} with elements of serialized by kryo. */
32+
public class TtlListStateWithKryoTestContext
33+
extends TtlListStateTestContextBase<TtlListStateWithKryoTestContext.NotPojoElement> {
34+
TtlListStateWithKryoTestContext() {
35+
super(new KryoSerializer<>(NotPojoElement.class, getForceKryoSerializerConfig()));
36+
}
37+
38+
private static SerializerConfig getForceKryoSerializerConfig() {
39+
Configuration config = new Configuration();
40+
config.set(PipelineOptions.FORCE_KRYO, true);
41+
return new SerializerConfigImpl(config, new ExecutionConfig(config));
42+
}
43+
44+
@Override
45+
NotPojoElement generateRandomElement(int i) {
46+
return new NotPojoElement(RANDOM.nextInt(100));
47+
}
48+
49+
@Override
50+
void initTestValues() {
51+
emptyValue = Collections.emptyList();
52+
53+
updateEmpty =
54+
Arrays.asList(new NotPojoElement(5), new NotPojoElement(7), new NotPojoElement(10));
55+
updateUnexpired =
56+
Arrays.asList(new NotPojoElement(8), new NotPojoElement(9), new NotPojoElement(11));
57+
updateExpired = Arrays.asList(new NotPojoElement(1), new NotPojoElement(4));
58+
59+
getUpdateEmpty = updateEmpty;
60+
getUnexpired = updateUnexpired;
61+
getUpdateExpired = updateExpired;
62+
}
63+
64+
public static class NotPojoElement {
65+
public int value;
66+
67+
public NotPojoElement(int value) {
68+
this.value = value;
69+
}
70+
71+
@Override
72+
public String toString() {
73+
return "NotPojoElement{" + "value=" + value + '}';
74+
}
75+
76+
@Override
77+
public boolean equals(Object obj) {
78+
if (this == obj) {
79+
return true;
80+
}
81+
if (obj == null || getClass() != obj.getClass()) {
82+
return false;
83+
}
84+
NotPojoElement that = (NotPojoElement) obj;
85+
return value == that.value;
86+
}
87+
88+
@Override
89+
public int hashCode() {
90+
return Integer.hashCode(value);
91+
}
92+
}
93+
}

flink-runtime/src/test/java/org/apache/flink/runtime/state/ttl/TtlStateTestBase.java

Lines changed: 1 addition & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -87,6 +87,7 @@ protected abstract StateBackendTestContext createStateBackendTestContext(
8787
new TtlValueStateTestContext(),
8888
new TtlFixedLenElemListStateTestContext(),
8989
new TtlNonFixedLenElemListStateTestContext(),
90+
new TtlListStateWithKryoTestContext(),
9091
new TtlMapStateAllEntriesTestContext(),
9192
new TtlMapStatePerElementTestContext(),
9293
new TtlMapStatePerNullElementTestContext(),

flink-state-backends/flink-statebackend-rocksdb/src/main/java/org/apache/flink/contrib/streaming/state/ttl/RocksDbTtlCompactFiltersManager.java

Lines changed: 55 additions & 9 deletions
Original file line numberDiff line numberDiff line change
@@ -47,6 +47,7 @@
4747

4848
import java.io.IOException;
4949
import java.util.LinkedHashMap;
50+
import java.util.function.Supplier;
5051

5152
/** RocksDB compaction filter utils for state with TTL. */
5253
public class RocksDbTtlCompactFiltersManager {
@@ -160,15 +161,27 @@ public void configCompactFilter(
160161

161162
private static class ListElementFilterFactory<T>
162163
implements FlinkCompactionFilter.ListElementFilterFactory {
163-
private final TypeSerializer<T> serializer;
164+
// {@See #createListElementFilter}.
165+
private final ThreadLocalSerializerProvider<T> threadLocalSerializer;
164166

165167
private ListElementFilterFactory(TypeSerializer<T> serializer) {
166-
this.serializer = serializer;
168+
ClassLoader contextClassLoader = null;
169+
try {
170+
contextClassLoader = Thread.currentThread().getContextClassLoader();
171+
} catch (Throwable e) {
172+
LOG.info("Cannot get context classloader for list state's compaction filter.", e);
173+
}
174+
threadLocalSerializer =
175+
new ThreadLocalSerializerProvider<>(serializer, contextClassLoader);
167176
}
168177

169178
@Override
170179
public FlinkCompactionFilter.ListElementFilter createListElementFilter() {
171-
return new ListElementFilter<>(serializer);
180+
// This method will be invoked by native code multiple times when creating compaction
181+
// filter. And the created filter will be shared by multiple background threads.
182+
// Make sure the serializer is thread-local and has classloader set for each thread
183+
// correctly and individually.
184+
return new ListElementFilter<>(threadLocalSerializer);
172185
}
173186
}
174187

@@ -186,21 +199,22 @@ public long currentTimestamp() {
186199
}
187200

188201
private static class ListElementFilter<T> implements FlinkCompactionFilter.ListElementFilter {
189-
private final TypeSerializer<T> serializer;
190-
private DataInputDeserializer input;
202+
private final ThreadLocalSerializerProvider<T> threadLocalSerializer;
203+
private final DataInputDeserializer input;
191204

192-
private ListElementFilter(TypeSerializer<T> serializer) {
193-
this.serializer = serializer;
205+
private ListElementFilter(ThreadLocalSerializerProvider<T> serializer) {
206+
this.threadLocalSerializer = serializer;
194207
this.input = new DataInputDeserializer();
195208
}
196209

197210
@Override
198211
public int nextUnexpiredOffset(byte[] bytes, long ttl, long currentTimestamp) {
199212
input.setBuffer(bytes);
200213
int lastElementOffset = 0;
214+
TypeSerializer<T> serializer = threadLocalSerializer.get();
201215
while (input.available() > 0) {
202216
try {
203-
long timestamp = nextElementLastAccessTimestamp();
217+
long timestamp = nextElementLastAccessTimestamp(serializer);
204218
if (!TtlUtils.expired(timestamp, ttl, currentTimestamp)) {
205219
break;
206220
}
@@ -213,7 +227,8 @@ public int nextUnexpiredOffset(byte[] bytes, long ttl, long currentTimestamp) {
213227
return lastElementOffset;
214228
}
215229

216-
private long nextElementLastAccessTimestamp() throws IOException {
230+
private long nextElementLastAccessTimestamp(TypeSerializer<T> serializer)
231+
throws IOException {
217232
TtlValue<?> ttlValue = (TtlValue<?>) serializer.deserialize(input);
218233
if (input.available() > 0) {
219234
input.skipBytesToRead(1);
@@ -222,6 +237,37 @@ private long nextElementLastAccessTimestamp() throws IOException {
222237
}
223238
}
224239

240+
private static class ThreadLocalSerializerProvider<T> implements Supplier<TypeSerializer<T>> {
241+
// Multiple background threads may share the same filter instance, so we need to make sure
242+
// the serializer is thread-local, and every thread has its own instance with classloader.
243+
private final ThreadLocal<TypeSerializer<T>> threadLocalSerializer;
244+
245+
public ThreadLocalSerializerProvider(
246+
TypeSerializer<T> serializer, ClassLoader classLoader) {
247+
this.threadLocalSerializer =
248+
ThreadLocal.withInitial(
249+
() -> {
250+
setClassloaderIfNeeded(classLoader);
251+
return serializer.duplicate();
252+
});
253+
}
254+
255+
private void setClassloaderIfNeeded(ClassLoader classLoader) {
256+
// The classloader that should be set to the current thread when deserializing.
257+
// The reason why we should set classloader is that the serializer may be Kryo
258+
// serializer which needs user classloader to load user classes.
259+
// See FLINK-16686 for more details.
260+
if (classLoader != null) {
261+
Thread.currentThread().setContextClassLoader(classLoader);
262+
}
263+
}
264+
265+
@Override
266+
public TypeSerializer<T> get() {
267+
return threadLocalSerializer.get();
268+
}
269+
}
270+
225271
public void disposeAndClearRegisteredCompactionFactories() {
226272
for (FlinkCompactionFilterFactory factory : compactionFilterFactories.values()) {
227273
IOUtils.closeQuietly(factory);

0 commit comments

Comments
 (0)