pnowojski commented on code in PR #20151: URL: https://github.com/apache/flink/pull/20151#discussion_r1000549678
########## flink-runtime/src/main/java/org/apache/flink/runtime/checkpoint/channel/ChannelStateWriteRequest.java: ########## @@ -205,9 +207,17 @@ public CheckpointStorageLocationReference getLocationReference() { return locationReference; } + public void registerCancelCallback(Consumer<Throwable> cancelCallback) { + this.cancelCallback = cancelCallback; + } + @Override public void cancel(Throwable cause) { - targetResult.fail(cause); + if (cancelCallback == null) { + targetResult.fail(cause); + return; + } + cancelCallback.accept(cause); Review Comment: why we are not failing the `targetResult` if the `cancelCallback` is set? This seems strange. ########## flink-streaming-java/src/main/java/org/apache/flink/streaming/api/environment/ExecutionCheckpointingOptions.java: ########## @@ -252,4 +252,13 @@ public class ExecutionCheckpointingOptions { "{{.Site.BaseURL}}{{.Site.LanguagePrefix}}/docs/dev/datastream/fault-tolerance/checkpointing/#checkpointing-with-parts-of-the-graph-finished-beta", "the important considerations")) .build()); + + public static final ConfigOption<Integer> CHANNEL_STATE_NUMBER_OF_TASKS_SHARE_FILE = + ConfigOptions.key("execution.checkpointing.channel-state.number-of-tasks-share-file") + .intType() + .defaultValue(5) + .withDescription( + "Defines the maximum number of tasks that share the same channel state file. " + + "It can reduce the number of small files when enable unaligned checkpoint. " + + "Each task will create a new channel state file when this is configured to 1."); Review Comment: `task` -> `subtask` -- This is an automated message from the Apache Git Service. To respond to the message, please log on to GitHub and use the URL above to go to the specific comment. To unsubscribe, e-mail: issues-unsubscr...@flink.apache.org For queries about this service, please contact Infrastructure at: us...@infra.apache.org