pnowojski commented on a change in pull request #13234:
URL: https://github.com/apache/flink/pull/13234#discussion_r478509078



##########
File path: 
flink-streaming-java/src/main/java/org/apache/flink/streaming/runtime/tasks/OperatorChain.java
##########
@@ -208,19 +206,100 @@ public OperatorChain(
        OperatorChain(
                        List<StreamOperatorWrapper<?, ?>> allOperatorWrappers,
                        RecordWriterOutput<?>[] streamOutputs,
-                       WatermarkGaugeExposingOutput<StreamRecord<OUT>> 
chainEntryPoint,
-                       StreamOperatorWrapper<OUT, OP> headOperatorWrapper) {
+                       WatermarkGaugeExposingOutput<StreamRecord<OUT>> 
mainOperatorOutput,
+                       StreamOperatorWrapper<OUT, OP> mainOperatorWrapper) {
 
                this.streamOutputs = checkNotNull(streamOutputs);
-               this.chainEntryPoint = checkNotNull(chainEntryPoint);
+               this.mainOperatorOutput = checkNotNull(mainOperatorOutput);
                this.operatorEventDispatcher = null;
 
                checkState(allOperatorWrappers != null && 
allOperatorWrappers.size() > 0);
-               this.headOperatorWrapper = checkNotNull(headOperatorWrapper);
+               this.mainOperatorWrapper = checkNotNull(mainOperatorWrapper);
                this.tailOperatorWrapper = allOperatorWrappers.get(0);
                this.numOperators = allOperatorWrappers.size();
+               this.chainedSourceOutputs = new ChainedSourceOutputs();
 
-               linkOperatorWrappers(allOperatorWrappers);
+               firstOperatorWrapper = 
linkOperatorWrappers(allOperatorWrappers);
+       }
+
+       private void createChainOutputs(
+               List<StreamEdge> outEdgesInOrder,
+               RecordWriterDelegate<SerializationDelegate<StreamRecord<OUT>>> 
recordWriterDelegate,
+               Map<Integer, StreamConfig> chainedConfigs,
+               StreamTask<OUT, OP> containingTask,
+               Map<StreamEdge, RecordWriterOutput<?>> streamOutputMap) {
+               for (int i = 0; i < outEdgesInOrder.size(); i++) {
+                       StreamEdge outEdge = outEdgesInOrder.get(i);
+
+                       RecordWriterOutput<?> streamOutput = createStreamOutput(
+                               recordWriterDelegate.getRecordWriter(i),
+                               outEdge,
+                               chainedConfigs.get(outEdge.getSourceId()),
+                               containingTask.getEnvironment());
+
+                       this.streamOutputs[i] = streamOutput;
+                       streamOutputMap.put(outEdge, streamOutput);
+               }
+       }
+
+       private ChainedSourceOutputs createChainedInputs(
+                       StreamTask<OUT, OP> containingTask,
+                       StreamConfig.Input[] configuredInputs,
+                       Map<Integer, StreamConfig> chainedConfigs,
+                       ClassLoader userCodeClassloader,
+                       List<StreamOperatorWrapper<?, ?>> allOpWrappers) {
+               if (Arrays.stream(configuredInputs).noneMatch(input -> input 
instanceof SourceInput)) {
+                       return new ChainedSourceOutputs();
+               }
+               checkState(
+                       mainOperatorWrapper.getStreamOperator() instanceof 
MultipleInputStreamOperator,

Review comment:
       It is, but it would be an unsupported configuration. If there are 
chained sources, it's currently only supported with 
`MultipleInputStreamOperator` - not with for example `TwoInputStreamOperator`.




----------------------------------------------------------------
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.

For queries about this service, please contact Infrastructure at:
[email protected]


Reply via email to