This is an automated email from the ASF dual-hosted git repository. zhuzh pushed a commit to branch master in repository https://gitbox.apache.org/repos/asf/flink.git
commit 401aefdcf16992bde636eb617decdfa1ffd36622 Author: ifndef-SleePy <mmyy1...@gmail.com> AuthorDate: Mon Jan 30 18:06:45 2023 +0800 [FLINK-30823][runtime] Enable speculative execution for FileSink --- .../src/main/java/org/apache/flink/connector/file/sink/FileSink.java | 4 +++- 1 file changed, 3 insertions(+), 1 deletion(-) diff --git a/flink-connectors/flink-connector-files/src/main/java/org/apache/flink/connector/file/sink/FileSink.java b/flink-connectors/flink-connector-files/src/main/java/org/apache/flink/connector/file/sink/FileSink.java index 236582767e3..f7ef366d50c 100644 --- a/flink-connectors/flink-connector-files/src/main/java/org/apache/flink/connector/file/sink/FileSink.java +++ b/flink-connectors/flink-connector-files/src/main/java/org/apache/flink/connector/file/sink/FileSink.java @@ -21,6 +21,7 @@ package org.apache.flink.connector.file.sink; import org.apache.flink.annotation.Experimental; import org.apache.flink.annotation.Internal; import org.apache.flink.annotation.PublicEvolving; +import org.apache.flink.api.common.SupportsConcurrentExecutionAttempts; import org.apache.flink.api.common.serialization.BulkWriter; import org.apache.flink.api.common.serialization.Encoder; import org.apache.flink.api.common.typeinfo.TypeInformation; @@ -130,7 +131,8 @@ public class FileSink<IN> implements StatefulSink<IN, FileWriterBucketState>, TwoPhaseCommittingSink<IN, FileSinkCommittable>, WithCompatibleState, - WithPreCommitTopology<IN, FileSinkCommittable> { + WithPreCommitTopology<IN, FileSinkCommittable>, + SupportsConcurrentExecutionAttempts { private final BucketsBuilder<IN, ? extends BucketsBuilder<IN, ?>> bucketsBuilder;