From 9054b85961f6822125c6a45729869d9285247ab4 Mon Sep 17 00:00:00 2001 From: ForwardXu Date: Mon, 25 Apr 2022 12:58:27 +0800 Subject: [PATCH] Revert "[HUDI-3951]support generan parameter 'sink.parallelism' for flink-hudi (#5405)" (#5421) This reverts commit bda3db078e927421c10932cfcb3019cfddb125b6. --- .../main/java/org/apache/hudi/configuration/FlinkOptions.java | 3 --- .../src/main/java/org/apache/hudi/table/HoodieTableSink.java | 3 +-- 2 files changed, 1 insertion(+), 5 deletions(-) diff --git a/hudi-flink-datasource/hudi-flink/src/main/java/org/apache/hudi/configuration/FlinkOptions.java b/hudi-flink-datasource/hudi-flink/src/main/java/org/apache/hudi/configuration/FlinkOptions.java index c6b16e6ec..e2be7d364 100644 --- a/hudi-flink-datasource/hudi-flink/src/main/java/org/apache/hudi/configuration/FlinkOptions.java +++ b/hudi-flink-datasource/hudi-flink/src/main/java/org/apache/hudi/configuration/FlinkOptions.java @@ -34,7 +34,6 @@ import org.apache.hudi.keygen.constant.KeyGeneratorType; import org.apache.flink.configuration.ConfigOption; import org.apache.flink.configuration.ConfigOptions; import org.apache.flink.configuration.Configuration; -import org.apache.flink.table.factories.FactoryUtil; import java.lang.reflect.Field; import java.util.ArrayList; @@ -233,8 +232,6 @@ public class FlinkOptions extends HoodieConfig { // ------------------------------------------------------------------------ // Write Options // ------------------------------------------------------------------------ - public static final ConfigOption SINK_PARALLELISM = FactoryUtil.SINK_PARALLELISM; - public static final ConfigOption TABLE_NAME = ConfigOptions .key(HoodieWriteConfig.TBL_NAME.key()) .stringType() diff --git a/hudi-flink-datasource/hudi-flink/src/main/java/org/apache/hudi/table/HoodieTableSink.java b/hudi-flink-datasource/hudi-flink/src/main/java/org/apache/hudi/table/HoodieTableSink.java index 46e360d3a..4dd4f89d0 100644 --- a/hudi-flink-datasource/hudi-flink/src/main/java/org/apache/hudi/table/HoodieTableSink.java +++ b/hudi-flink-datasource/hudi-flink/src/main/java/org/apache/hudi/table/HoodieTableSink.java @@ -82,8 +82,7 @@ public class HoodieTableSink implements DynamicTableSink, SupportsPartitioning, } // default parallelism - int parallelism = conf.getInteger(FlinkOptions.SINK_PARALLELISM, - dataStream.getExecutionConfig().getParallelism()); + int parallelism = dataStream.getExecutionConfig().getParallelism(); DataStream pipeline; // bootstrap final DataStream hoodieRecordDataStream =