[HUDI-2121] Add operator uid for flink stateful operators (#3212)
This commit is contained in:
@@ -80,9 +80,11 @@ public class HoodieTableSink implements DynamicTableSink, SupportsPartitioning,
|
||||
|
||||
// TODO: This is a very time-consuming operation, will optimization
|
||||
if (conf.getBoolean(FlinkOptions.INDEX_BOOTSTRAP_ENABLED)) {
|
||||
hoodieDataStream = hoodieDataStream.rebalance().transform("index_bootstrap",
|
||||
TypeInformation.of(HoodieRecord.class),
|
||||
new ProcessOperator<>(new BootstrapFunction<>(conf)));
|
||||
hoodieDataStream = hoodieDataStream.rebalance()
|
||||
.transform("index_bootstrap",
|
||||
TypeInformation.of(HoodieRecord.class),
|
||||
new ProcessOperator<>(new BootstrapFunction<>(conf)))
|
||||
.uid("uid_index_bootstrap_" + conf.getString(FlinkOptions.TABLE_NAME));
|
||||
}
|
||||
|
||||
DataStream<Object> pipeline = hoodieDataStream
|
||||
|
||||
@@ -179,8 +179,10 @@ public class HoodieTableSource implements
|
||||
}
|
||||
OneInputStreamOperatorFactory<MergeOnReadInputSplit, RowData> factory = StreamReadOperator.factory((MergeOnReadInputFormat) inputFormat);
|
||||
SingleOutputStreamOperator<RowData> source = execEnv.addSource(monitoringFunction, "streaming_source")
|
||||
.uid("uid_streaming_source_" + conf.getString(FlinkOptions.TABLE_NAME))
|
||||
.setParallelism(1)
|
||||
.transform("split_reader", typeInfo, factory)
|
||||
.uid("uid_split_reader_" + conf.getString(FlinkOptions.TABLE_NAME))
|
||||
.setParallelism(conf.getInteger(FlinkOptions.READ_TASKS));
|
||||
return new DataStreamSource<>(source);
|
||||
} else {
|
||||
|
||||
Reference in New Issue
Block a user