Skip to content

Commit

Permalink
[hotfix][task] Rename SubtaskCheckpointCoordinator#executorService to…
Browse files Browse the repository at this point in the history
… asyncOperationsThreadPool
  • Loading branch information
pnowojski committed Oct 13, 2020
1 parent 5a7dbcd commit 0ee3a5f
Showing 1 changed file with 8 additions and 8 deletions.
Original file line number Diff line number Diff line change
Expand Up @@ -76,7 +76,7 @@ class SubtaskCheckpointCoordinatorImpl implements SubtaskCheckpointCoordinator {

private final CachingCheckpointStorageWorkerView checkpointStorage;
private final String taskName;
private final ExecutorService executorService;
private final ExecutorService asyncOperationsThreadPool;
private final Environment env;
private final AsyncExceptionHandler asyncExceptionHandler;
private final ChannelStateWriter channelStateWriter;
Expand All @@ -101,7 +101,7 @@ class SubtaskCheckpointCoordinatorImpl implements SubtaskCheckpointCoordinator {
String taskName,
StreamTaskActionExecutor actionExecutor,
CloseableRegistry closeableRegistry,
ExecutorService executorService,
ExecutorService asyncOperationsThreadPool,
Environment env,
AsyncExceptionHandler asyncExceptionHandler,
boolean unalignedCheckpointEnabled,
Expand All @@ -110,7 +110,7 @@ class SubtaskCheckpointCoordinatorImpl implements SubtaskCheckpointCoordinator {
taskName,
actionExecutor,
closeableRegistry,
executorService,
asyncOperationsThreadPool,
env,
asyncExceptionHandler,
unalignedCheckpointEnabled,
Expand All @@ -123,7 +123,7 @@ class SubtaskCheckpointCoordinatorImpl implements SubtaskCheckpointCoordinator {
String taskName,
StreamTaskActionExecutor actionExecutor,
CloseableRegistry closeableRegistry,
ExecutorService executorService,
ExecutorService asyncOperationsThreadPool,
Environment env,
AsyncExceptionHandler asyncExceptionHandler,
boolean unalignedCheckpointEnabled,
Expand All @@ -134,7 +134,7 @@ class SubtaskCheckpointCoordinatorImpl implements SubtaskCheckpointCoordinator {
taskName,
actionExecutor,
closeableRegistry,
executorService,
asyncOperationsThreadPool,
env,
asyncExceptionHandler,
prepareInputSnapshot,
Expand All @@ -148,7 +148,7 @@ class SubtaskCheckpointCoordinatorImpl implements SubtaskCheckpointCoordinator {
String taskName,
StreamTaskActionExecutor actionExecutor,
CloseableRegistry closeableRegistry,
ExecutorService executorService,
ExecutorService asyncOperationsThreadPool,
Environment env,
AsyncExceptionHandler asyncExceptionHandler,
BiFunctionWithException<ChannelStateWriter, Long, CompletableFuture<Void>, IOException> prepareInputSnapshot,
Expand All @@ -158,7 +158,7 @@ class SubtaskCheckpointCoordinatorImpl implements SubtaskCheckpointCoordinator {
this.taskName = checkNotNull(taskName);
this.checkpoints = new HashMap<>();
this.lock = new Object();
this.executorService = checkNotNull(executorService);
this.asyncOperationsThreadPool = checkNotNull(asyncOperationsThreadPool);
this.env = checkNotNull(env);
this.asyncExceptionHandler = checkNotNull(asyncExceptionHandler);
this.actionExecutor = checkNotNull(actionExecutor);
Expand Down Expand Up @@ -437,7 +437,7 @@ private void prepareInflightDataSnapshot(long checkpointId) throws IOException {

private void finishAndReportAsync(Map<OperatorID, OperatorSnapshotFutures> snapshotFutures, CheckpointMetaData metadata, CheckpointMetrics metrics, CheckpointOptions options) {
// we are transferring ownership over snapshotInProgressList for cleanup to the thread, active on submit
executorService.execute(new AsyncCheckpointRunnable(
asyncOperationsThreadPool.execute(new AsyncCheckpointRunnable(
snapshotFutures,
metadata,
metrics,
Expand Down

0 comments on commit 0ee3a5f

Please sign in to comment.