5555import java .util .concurrent .ConcurrentHashMap ;
5656import java .util .concurrent .ExecutorService ;
5757import java .util .concurrent .Executors ;
58- import java .util .concurrent .atomic .AtomicReference ;
5958import java .util .stream .Collectors ;
6059import java .util .stream .Stream ;
6160
@@ -78,7 +77,10 @@ public class SchemaRegistryRequestHandler implements Closeable {
7877 /**
7978 * Atomic flag indicating if current RequestHandler could accept more schema changes for now.
8079 */
81- private final AtomicReference <RequestStatus > schemaChangeStatus ;
80+ private volatile RequestStatus schemaChangeStatus ;
81+
82+ private final List <Integer > pendingSubTaskIds ;
83+ private final Object schemaChangeRequestLock ;
8284
8385 private volatile Throwable currentChangeException ;
8486 private volatile List <SchemaChangeEvent > currentDerivedSchemaChangeEvents ;
@@ -110,7 +112,10 @@ public SchemaRegistryRequestHandler(
110112 this .currentDerivedSchemaChangeEvents = new ArrayList <>();
111113 this .currentFinishedSchemaChanges = new ArrayList <>();
112114 this .currentIgnoredSchemaChanges = new ArrayList <>();
113- this .schemaChangeStatus = new AtomicReference <>(RequestStatus .IDLE );
115+
116+ this .schemaChangeStatus = RequestStatus .IDLE ;
117+ this .pendingSubTaskIds = new ArrayList <>();
118+ this .schemaChangeRequestLock = new Object ();
114119 }
115120
116121 /**
@@ -120,67 +125,100 @@ public SchemaRegistryRequestHandler(
120125 */
121126 public CompletableFuture <CoordinationResponse > handleSchemaChangeRequest (
122127 SchemaChangeRequest request ) {
123- if (schemaChangeStatus .compareAndSet (RequestStatus .IDLE , RequestStatus .WAITING_FOR_FLUSH )) {
124- LOG .info (
125- "Received schema change event request {} from table {}. SchemaChangeStatus switched from IDLE to WAITING_FOR_FLUSH, other requests will be blocked." ,
126- request .getSchemaChangeEvent (),
127- request .getTableId ().toString ());
128- SchemaChangeEvent event = request .getSchemaChangeEvent ();
129-
130- // If this schema change event has been requested by another subTask, ignore it.
131- if (schemaManager .isOriginalSchemaChangeEventRedundant (event )) {
132- LOG .info ("Event {} has been addressed before, ignoring it." , event );
133- clearCurrentSchemaChangeRequest ();
134- Preconditions .checkState (
135- schemaChangeStatus .compareAndSet (
136- RequestStatus .WAITING_FOR_FLUSH , RequestStatus .IDLE ),
137- "Illegal schemaChangeStatus state: should still in WAITING_FOR_FLUSH state if event was duplicated, not "
138- + schemaChangeStatus .get ());
128+
129+ // We use requester subTask ID as the pending ticket, because there will be at most 1 schema
130+ // change requests simultaneously from each subTask
131+ int requestSubTaskId = request .getSubTaskId ();
132+
133+ synchronized (schemaChangeRequestLock ) {
134+ // Make sure we handle the first request in the pending list to avoid out-of-order
135+ // waiting and blocks checkpointing mechanism.
136+ if (schemaChangeStatus == RequestStatus .IDLE ) {
137+ if (pendingSubTaskIds .isEmpty ()) {
138+ LOG .info (
139+ "Received schema change event request {} from table {} from subTask {}. Pending list is empty, handling this." ,
140+ request .getSchemaChangeEvent (),
141+ request .getTableId ().toString (),
142+ requestSubTaskId );
143+ } else if (pendingSubTaskIds .get (0 ) == requestSubTaskId ) {
144+ LOG .info (
145+ "Received schema change event request {} from table {} from subTask {}. It is on the first of the pending list, handling this." ,
146+ request .getSchemaChangeEvent (),
147+ request .getTableId ().toString (),
148+ requestSubTaskId );
149+ pendingSubTaskIds .remove (0 );
150+ } else {
151+ LOG .info (
152+ "Received schema change event request {} from table {} from subTask {}. It is not the first of the pending list ({})." ,
153+ request .getSchemaChangeEvent (),
154+ request .getTableId ().toString (),
155+ requestSubTaskId ,
156+ pendingSubTaskIds );
157+ if (!pendingSubTaskIds .contains (requestSubTaskId )) {
158+ pendingSubTaskIds .add (requestSubTaskId );
159+ }
160+ return CompletableFuture .completedFuture (wrap (SchemaChangeResponse .busy ()));
161+ }
162+
163+ SchemaChangeEvent event = request .getSchemaChangeEvent ();
164+
165+ // If this schema change event has been requested by another subTask, ignore it.
166+ if (schemaManager .isOriginalSchemaChangeEventRedundant (event )) {
167+ LOG .info ("Event {} has been addressed before, ignoring it." , event );
168+ clearCurrentSchemaChangeRequest ();
169+ LOG .info (
170+ "SchemaChangeStatus switched from WAITING_FOR_FLUSH to IDLE for request {} due to duplicated request." ,
171+ request );
172+ return CompletableFuture .completedFuture (
173+ wrap (SchemaChangeResponse .duplicate ()));
174+ }
175+ schemaManager .applyOriginalSchemaChange (event );
176+ List <SchemaChangeEvent > derivedSchemaChangeEvents =
177+ calculateDerivedSchemaChangeEvents (request .getSchemaChangeEvent ());
178+
179+ // If this schema change event is filtered out by LENIENT mode or merging table
180+ // route strategies, ignore it.
181+ if (derivedSchemaChangeEvents .isEmpty ()) {
182+ LOG .info ("Event {} is omitted from sending to downstream, ignoring it." , event );
183+ clearCurrentSchemaChangeRequest ();
184+ LOG .info (
185+ "SchemaChangeStatus switched from WAITING_FOR_FLUSH to IDLE for request {} due to ignored request." ,
186+ request );
187+ return CompletableFuture .completedFuture (wrap (SchemaChangeResponse .ignored ()));
188+ }
189+
139190 LOG .info (
140- "SchemaChangeStatus switched from WAITING_FOR_FLUSH to IDLE for request {} due to duplicated request." ,
141- request );
142- return CompletableFuture .completedFuture (wrap (SchemaChangeResponse .duplicate ()));
143- }
144- schemaManager .applyOriginalSchemaChange (event );
145- List <SchemaChangeEvent > derivedSchemaChangeEvents =
146- calculateDerivedSchemaChangeEvents (request .getSchemaChangeEvent ());
147-
148- // If this schema change event is filtered out by LENIENT mode or merging table route
149- // strategies, ignore it.
150- if (derivedSchemaChangeEvents .isEmpty ()) {
151- LOG .info ("Event {} is omitted from sending to downstream, ignoring it." , event );
152- clearCurrentSchemaChangeRequest ();
153- Preconditions .checkState (
154- schemaChangeStatus .compareAndSet (
155- RequestStatus .WAITING_FOR_FLUSH , RequestStatus .IDLE ),
156- "Illegal schemaChangeStatus state: should still in WAITING_FOR_FLUSH state if event was ignored, not "
157- + schemaChangeStatus .get ());
191+ "SchemaChangeStatus switched from IDLE to WAITING_FOR_FLUSH, other requests will be blocked." );
192+ // This request has been accepted.
193+ schemaChangeStatus = RequestStatus .WAITING_FOR_FLUSH ;
194+
195+ // Backfill pre-schema info for sink applying
196+ derivedSchemaChangeEvents .forEach (
197+ e -> {
198+ if (e instanceof SchemaChangeEventWithPreSchema ) {
199+ SchemaChangeEventWithPreSchema pe =
200+ (SchemaChangeEventWithPreSchema ) e ;
201+ if (!pe .hasPreSchema ()) {
202+ schemaManager
203+ .getLatestEvolvedSchema (pe .tableId ())
204+ .ifPresent (pe ::fillPreSchema );
205+ }
206+ }
207+ });
208+ currentDerivedSchemaChangeEvents = new ArrayList <>(derivedSchemaChangeEvents );
209+ return CompletableFuture .completedFuture (
210+ wrap (SchemaChangeResponse .accepted (derivedSchemaChangeEvents )));
211+ } else {
158212 LOG .info (
159- "SchemaChangeStatus switched from WAITING_FOR_FLUSH to IDLE for request {} due to ignored request." ,
160- request );
161- return CompletableFuture .completedFuture (wrap (SchemaChangeResponse .ignored ()));
213+ "Schema Registry is busy processing a schema change request, could not handle request {} for now. Added {} to pending list ({})." ,
214+ request ,
215+ requestSubTaskId ,
216+ pendingSubTaskIds );
217+ if (!pendingSubTaskIds .contains (requestSubTaskId )) {
218+ pendingSubTaskIds .add (requestSubTaskId );
219+ }
220+ return CompletableFuture .completedFuture (wrap (SchemaChangeResponse .busy ()));
162221 }
163-
164- // Backfill pre-schema info for sink applying
165- derivedSchemaChangeEvents .forEach (
166- e -> {
167- if (e instanceof SchemaChangeEventWithPreSchema ) {
168- SchemaChangeEventWithPreSchema pe = (SchemaChangeEventWithPreSchema ) e ;
169- if (!pe .hasPreSchema ()) {
170- schemaManager
171- .getLatestEvolvedSchema (pe .tableId ())
172- .ifPresent (pe ::fillPreSchema );
173- }
174- }
175- });
176- currentDerivedSchemaChangeEvents = new ArrayList <>(derivedSchemaChangeEvents );
177- return CompletableFuture .completedFuture (
178- wrap (SchemaChangeResponse .accepted (derivedSchemaChangeEvents )));
179- } else {
180- LOG .info (
181- "Schema Registry is busy processing a schema change request, could not handle request {} for now." ,
182- request );
183- return CompletableFuture .completedFuture (wrap (SchemaChangeResponse .busy ()));
184222 }
185223 }
186224
@@ -227,9 +265,10 @@ private void applySchemaChange(
227265 }
228266 }
229267 Preconditions .checkState (
230- schemaChangeStatus . compareAndSet ( RequestStatus . APPLYING , RequestStatus .FINISHED ) ,
268+ schemaChangeStatus == RequestStatus .APPLYING ,
231269 "Illegal schemaChangeStatus state: should be APPLYING before applySchemaChange finishes, not "
232- + schemaChangeStatus .get ());
270+ + schemaChangeStatus );
271+ schemaChangeStatus = RequestStatus .FINISHED ;
233272 LOG .info (
234273 "SchemaChangeStatus switched from APPLYING to FINISHED for request {}." ,
235274 currentDerivedSchemaChangeEvents );
@@ -262,10 +301,11 @@ public void flushSuccess(TableId tableId, int sinkSubtask, int parallelism) {
262301 }
263302 if (flushedSinkWriters .equals (activeSinkWriters )) {
264303 Preconditions .checkState (
265- schemaChangeStatus .compareAndSet (
266- RequestStatus .WAITING_FOR_FLUSH , RequestStatus .APPLYING ),
304+ schemaChangeStatus == RequestStatus .WAITING_FOR_FLUSH ,
267305 "Illegal schemaChangeStatus state: should be WAITING_FOR_FLUSH before collecting enough FlushEvents, not "
268306 + schemaChangeStatus );
307+
308+ schemaChangeStatus = RequestStatus .APPLYING ;
269309 LOG .info (
270310 "All sink subtask have flushed for table {}. Start to apply schema change." ,
271311 tableId .toString ());
@@ -276,9 +316,10 @@ public void flushSuccess(TableId tableId, int sinkSubtask, int parallelism) {
276316
277317 public CompletableFuture <CoordinationResponse > getSchemaChangeResult () {
278318 Preconditions .checkState (
279- ! schemaChangeStatus . get (). equals ( RequestStatus .IDLE ) ,
319+ schemaChangeStatus != RequestStatus .IDLE ,
280320 "Illegal schemaChangeStatus: should not be IDLE before getting schema change request results." );
281- if (schemaChangeStatus .compareAndSet (RequestStatus .FINISHED , RequestStatus .IDLE )) {
321+ if (schemaChangeStatus == RequestStatus .FINISHED ) {
322+ schemaChangeStatus = RequestStatus .IDLE ;
282323 LOG .info (
283324 "SchemaChangeStatus switched from FINISHED to IDLE for request {}" ,
284325 currentDerivedSchemaChangeEvents );
0 commit comments