Interface TwoInputStreamOperator<IN1,IN2,OUT>
-
- Type Parameters:
IN1
- The input type of the operatorIN2
- The input type of the operatorOUT
- The output type of the operator
- All Superinterfaces:
CheckpointListener
,KeyContext
,Serializable
,StreamOperator<OUT>
- All Known Implementing Classes:
AbstractAsyncStateStreamingJoinOperator
,AbstractExternalTwoInputPythonFunctionOperator
,AbstractStreamingJoinOperator
,AbstractTwoInputEmbeddedPythonFunctionOperator
,AsyncStateStreamingJoinOperator
,BaseTwoInputStreamOperatorWithStateRetention
,BatchCoBroadcastWithKeyedOperator
,BatchCoBroadcastWithNonKeyedOperator
,CoBroadcastWithKeyedOperator
,CoBroadcastWithNonKeyedOperator
,CoProcessOperator
,CoStreamFlatMap
,CoStreamMap
,EmbeddedPythonBatchCoBroadcastProcessOperator
,EmbeddedPythonBatchKeyedCoBroadcastProcessOperator
,EmbeddedPythonCoProcessOperator
,EmbeddedPythonKeyedCoProcessOperator
,ExternalPythonBatchCoBroadcastProcessOperator
,ExternalPythonBatchKeyedCoBroadcastProcessOperator
,ExternalPythonCoProcessOperator
,ExternalPythonKeyedCoProcessOperator
,HashJoinOperator
,IntervalJoinOperator
,KeyedCoProcessOperator
,KeyedCoProcessOperatorWithWatermarkDelay
,KeyedTwoInputBroadcastProcessOperator
,KeyedTwoInputNonBroadcastProcessOperator
,LegacyKeyedCoProcessOperator
,MiniBatchStreamingJoinOperator
,SortMergeJoinOperator
,StreamingJoinOperator
,StreamingSemiAntiJoinOperator
,TemporalProcessTimeJoinOperator
,TemporalRowTimeJoinOperator
,TwoInputBroadcastProcessOperator
,TwoInputNonBroadcastProcessOperator
,WindowJoinOperator
@PublicEvolving public interface TwoInputStreamOperator<IN1,IN2,OUT> extends StreamOperator<OUT>
Interface for stream operators with two inputs. UseAbstractStreamOperator
as a base class if you want to implement a custom operator.
-
-
Method Summary
All Methods Instance Methods Abstract Methods Default Methods Modifier and Type Method Description void
processElement1(StreamRecord<IN1> element)
Processes one element that arrived on the first input of this two-input operator.void
processElement2(StreamRecord<IN2> element)
Processes one element that arrived on the second input of this two-input operator.void
processLatencyMarker1(LatencyMarker latencyMarker)
Processes aLatencyMarker
that arrived on the first input of this two-input operator.void
processLatencyMarker2(LatencyMarker latencyMarker)
Processes aLatencyMarker
that arrived on the second input of this two-input operator.default void
processRecordAttributes1(RecordAttributes recordAttributes)
Processes aRecordAttributes
that arrived on the first input of this operator.default void
processRecordAttributes2(RecordAttributes recordAttributes)
Processes aRecordAttributes
that arrived on the second input of this operator.void
processWatermark1(Watermark mark)
Processes aWatermark
that arrived on the first input of this two-input operator.void
processWatermark2(Watermark mark)
Processes aWatermark
that arrived on the second input of this two-input operator.void
processWatermarkStatus1(WatermarkStatus watermarkStatus)
Processes aWatermarkStatus
that arrived on the first input of this two-input operator.void
processWatermarkStatus2(WatermarkStatus watermarkStatus)
Processes aWatermarkStatus
that arrived on the second input of this two-input operator.-
Methods inherited from interface org.apache.flink.api.common.state.CheckpointListener
notifyCheckpointAborted, notifyCheckpointComplete
-
Methods inherited from interface org.apache.flink.streaming.api.operators.KeyContext
getCurrentKey, setCurrentKey
-
Methods inherited from interface org.apache.flink.streaming.api.operators.StreamOperator
close, finish, getMetricGroup, getOperatorAttributes, getOperatorID, initializeState, open, prepareSnapshotPreBarrier, setKeyContextElement1, setKeyContextElement2, snapshotState
-
-
-
-
Method Detail
-
processElement1
void processElement1(StreamRecord<IN1> element) throws Exception
Processes one element that arrived on the first input of this two-input operator. This method is guaranteed to not be called concurrently with other methods of the operator.- Throws:
Exception
-
processElement2
void processElement2(StreamRecord<IN2> element) throws Exception
Processes one element that arrived on the second input of this two-input operator. This method is guaranteed to not be called concurrently with other methods of the operator.- Throws:
Exception
-
processWatermark1
void processWatermark1(Watermark mark) throws Exception
Processes aWatermark
that arrived on the first input of this two-input operator. This method is guaranteed to not be called concurrently with other methods of the operator.
-
processWatermark2
void processWatermark2(Watermark mark) throws Exception
Processes aWatermark
that arrived on the second input of this two-input operator. This method is guaranteed to not be called concurrently with other methods of the operator.
-
processLatencyMarker1
void processLatencyMarker1(LatencyMarker latencyMarker) throws Exception
Processes aLatencyMarker
that arrived on the first input of this two-input operator. This method is guaranteed to not be called concurrently with other methods of the operator.- Throws:
Exception
- See Also:
LatencyMarker
-
processLatencyMarker2
void processLatencyMarker2(LatencyMarker latencyMarker) throws Exception
Processes aLatencyMarker
that arrived on the second input of this two-input operator. This method is guaranteed to not be called concurrently with other methods of the operator.- Throws:
Exception
- See Also:
LatencyMarker
-
processWatermarkStatus1
void processWatermarkStatus1(WatermarkStatus watermarkStatus) throws Exception
Processes aWatermarkStatus
that arrived on the first input of this two-input operator. This method is guaranteed to not be called concurrently with other methods of the operator.- Throws:
Exception
- See Also:
WatermarkStatus
-
processWatermarkStatus2
void processWatermarkStatus2(WatermarkStatus watermarkStatus) throws Exception
Processes aWatermarkStatus
that arrived on the second input of this two-input operator. This method is guaranteed to not be called concurrently with other methods of the operator.- Throws:
Exception
- See Also:
WatermarkStatus
-
processRecordAttributes1
@Experimental default void processRecordAttributes1(RecordAttributes recordAttributes) throws Exception
Processes aRecordAttributes
that arrived on the first input of this operator. This method is guaranteed to not be called concurrently with other methods of the operator.- Throws:
Exception
-
processRecordAttributes2
@Experimental default void processRecordAttributes2(RecordAttributes recordAttributes) throws Exception
Processes aRecordAttributes
that arrived on the second input of this operator. This method is guaranteed to not be called concurrently with other methods of the operator.- Throws:
Exception
-
-