Skip to content

Commit

Permalink
[cdc-base] Fix TM hangs caused by uncaught exception (apache#2511)
Browse files Browse the repository at this point in the history
  • Loading branch information
Jiabao-Sun authored Sep 21, 2023
1 parent 07db169 commit 7172695
Showing 1 changed file with 18 additions and 7 deletions.
Original file line number Diff line number Diff line change
Expand Up @@ -75,6 +75,8 @@ public IncrementalSourceScanFetcher(FetchTask.Context taskContext, int subtaskId
ThreadFactory threadFactory =
new ThreadFactoryBuilder()
.setNameFormat("debezium-snapshot-reader-" + subtaskId)
.setUncaughtExceptionHandler(
(thread, throwable) -> setReadException(throwable))
.build();
this.executorService = Executors.newSingleThreadExecutor(threadFactory);
this.hasNextElement = new AtomicBoolean(false);
Expand All @@ -89,17 +91,13 @@ public void submitTask(FetchTask<SourceSplitBase> fetchTask) {
this.queue = taskContext.getQueue();
this.hasNextElement.set(true);
this.reachEnd.set(false);
executorService.submit(

executorService.execute(
() -> {
try {
snapshotSplitReadTask.execute(taskContext);
} catch (Exception e) {
LOG.error(
String.format(
"Execute snapshot read task for snapshot split %s fail",
currentSnapshotSplit),
e);
readException = e;
setReadException(e);
}
});
}
Expand Down Expand Up @@ -186,6 +184,19 @@ private void checkReadException() {
}
}

private void setReadException(Throwable throwable) {
LOG.error(
String.format(
"Execute snapshot read task for snapshot split %s fail",
currentSnapshotSplit),
throwable);
if (readException == null) {
readException = throwable;
} else {
readException.addSuppressed(throwable);
}
}

@Override
public void close() {
try {
Expand Down

0 comments on commit 7172695

Please sign in to comment.