52
52
import java .util .concurrent .CompletableFuture ;
53
53
import java .util .concurrent .ConcurrentHashMap ;
54
54
import java .util .concurrent .ExecutorService ;
55
+ import java .util .concurrent .Executors ;
55
56
import java .util .concurrent .TimeoutException ;
56
57
import java .util .function .BooleanSupplier ;
57
58
@@ -96,6 +97,9 @@ public abstract class SchemaRegistry implements OperatorCoordinator, Coordinatio
96
97
protected transient SchemaManager schemaManager ;
97
98
protected transient TableIdRouter router ;
98
99
100
+ /** Executor service to execute handle event from operator. */
101
+ private final ExecutorService runInEventFromOperatorExecutor ;
102
+
99
103
protected SchemaRegistry (
100
104
OperatorCoordinator .Context context ,
101
105
String operatorName ,
@@ -111,6 +115,7 @@ protected SchemaRegistry(
111
115
this .routingRules = routingRules ;
112
116
this .rpcTimeout = rpcTimeout ;
113
117
this .behavior = schemaChangeBehavior ;
118
+ this .runInEventFromOperatorExecutor = Executors .newSingleThreadExecutor ();
114
119
}
115
120
116
121
// ---------------
@@ -130,6 +135,7 @@ public void start() throws Exception {
130
135
public void close () throws Exception {
131
136
LOG .info ("Closing SchemaRegistry - {}." , operatorName );
132
137
coordinatorExecutor .shutdown ();
138
+ runInEventFromOperatorExecutor .shutdown ();
133
139
}
134
140
135
141
// ------------------------------
@@ -235,6 +241,7 @@ public final CompletableFuture<CoordinationResponse> handleCoordinationRequest(
235
241
CoordinationRequest request ) {
236
242
CompletableFuture <CoordinationResponse > future = new CompletableFuture <>();
237
243
runInEventLoop (
244
+ coordinatorExecutor ,
238
245
() -> {
239
246
if (request instanceof GetEvolvedSchemaRequest ) {
240
247
handleGetEvolvedSchemaRequest ((GetEvolvedSchemaRequest ) request , future );
@@ -253,6 +260,7 @@ public final CompletableFuture<CoordinationResponse> handleCoordinationRequest(
253
260
public final void handleEventFromOperator (
254
261
int subTaskId , int attemptNumber , OperatorEvent event ) {
255
262
runInEventLoop (
263
+ runInEventFromOperatorExecutor ,
256
264
() -> {
257
265
if (event instanceof FlushSuccessEvent ) {
258
266
handleFlushSuccessEvent ((FlushSuccessEvent ) event );
@@ -297,7 +305,11 @@ public final void executionAttemptReady(
297
305
public final void checkpointCoordinator (
298
306
long checkpointId , CompletableFuture <byte []> completableFuture ) throws Exception {
299
307
LOG .info ("Going to start checkpoint No.{}" , checkpointId );
300
- runInEventLoop (() -> snapshot (completableFuture ), "Taking checkpoint - %d" , checkpointId );
308
+ runInEventLoop (
309
+ coordinatorExecutor ,
310
+ () -> snapshot (completableFuture ),
311
+ "Taking checkpoint - %d" ,
312
+ checkpointId );
301
313
}
302
314
303
315
@ Override
@@ -325,6 +337,7 @@ public final void resetToCheckpoint(long checkpointId, @Nullable byte[] checkpoi
325
337
* directly, make sure you're running heavy logics inside, or the entire job might hang!
326
338
*/
327
339
protected void runInEventLoop (
340
+ final ExecutorService coordinatorExecutor ,
328
341
final ThrowingRunnable <Throwable > action ,
329
342
final String actionName ,
330
343
final Object ... actionNameFormatParameters ) {
0 commit comments