Class CoFeedbackTransformation<F>
- java.lang.Object
-
- org.apache.flink.api.dag.Transformation<F>
-
- org.apache.flink.streaming.api.transformations.CoFeedbackTransformation<F>
-
- Type Parameters:
F
- The type of the feedback elements.
@Internal public class CoFeedbackTransformation<F> extends Transformation<F>
This represents a feedback point in a topology. The type of the feedback elements need not match the type of the upstreamTransformation
because the only allowed operations after aCoFeedbackTransformation
areTwoInputTransformations
. The upstreamTransformation
will be connected to the first input of the Co-Transform while the feedback edges will be connected to the second input.Both the partitioning of the input and the feedback edges is preserved. They can also have differing partitioning strategies. This requires, however, that the parallelism of the feedback
Transformations
must match the parallelism of the inputTransformation
.The upstream
Transformation
is not wired to thisCoFeedbackTransformation
. It is instead directly wired to theTwoInputTransformation
after thisCoFeedbackTransformation
.This is different from Iterations in batch processing.
- See Also:
FeedbackTransformation
-
-
Field Summary
-
Fields inherited from class org.apache.flink.api.dag.Transformation
bufferTimeout, description, id, name, outputType, typeUsed, UPPER_BOUND_MAX_PARALLELISM
-
-
Constructor Summary
Constructors Constructor Description CoFeedbackTransformation(int parallelism, TypeInformation<F> feedbackType, Long waitTime)
Creates a newCoFeedbackTransformation
from the given input.
-
Method Summary
All Methods Instance Methods Concrete Methods Modifier and Type Method Description void
addFeedbackEdge(Transformation<F> transform)
Adds a feedback edge.List<Transformation<F>>
getFeedbackEdges()
Returns the list of feedbackTransformations
.List<Transformation<?>>
getInputs()
Returns thetransformations
that are the immediate predecessors of the current transformation in the transformation graph.protected List<Transformation<?>>
getTransitivePredecessorsInternal()
Returns all transitive predecessorTransformation
s of thisTransformation
.Long
getWaitTime()
Returns the wait time.-
Methods inherited from class org.apache.flink.api.dag.Transformation
declareManagedMemoryUseCaseAtOperatorScope, declareManagedMemoryUseCaseAtSlotScope, equals, getAttribute, getBufferTimeout, getCoLocationGroupKey, getDescription, getId, getManagedMemoryOperatorScopeUseCaseWeights, getManagedMemorySlotScopeUseCases, getMaxParallelism, getMinResources, getName, getNewNodeId, getOutputType, getParallelism, getPreferredResources, getSlotSharingGroup, getTransitivePredecessors, getUid, getUserProvidedNodeHash, hashCode, isParallelismConfigured, setAttribute, setBufferTimeout, setCoLocationGroupKey, setDescription, setMaxParallelism, setName, setOutputType, setParallelism, setParallelism, setResources, setSlotSharingGroup, setSlotSharingGroup, setUid, setUidHash, toString, updateManagedMemoryStateBackendUseCase
-
-
-
-
Constructor Detail
-
CoFeedbackTransformation
public CoFeedbackTransformation(int parallelism, TypeInformation<F> feedbackType, Long waitTime)
Creates a newCoFeedbackTransformation
from the given input.- Parameters:
parallelism
- The parallelism of the upstreamTransformation
and the feedback edges.feedbackType
- The type of the feedback edgeswaitTime
- The wait time of the feedback operator. After the time expires the operation will close and not receive any more feedback elements.
-
-
Method Detail
-
addFeedbackEdge
public void addFeedbackEdge(Transformation<F> transform)
Adds a feedback edge. The parallelism of theTransformation
must match the parallelism of the inputTransformation
of the upstreamTransformation
.- Parameters:
transform
- The new feedbackTransformation
.
-
getFeedbackEdges
public List<Transformation<F>> getFeedbackEdges()
Returns the list of feedbackTransformations
.
-
getWaitTime
public Long getWaitTime()
Returns the wait time. This is the amount of time that the feedback operator keeps listening for feedback elements. Once the time expires the operation will close and will not receive further elements.
-
getTransitivePredecessorsInternal
protected List<Transformation<?>> getTransitivePredecessorsInternal()
Description copied from class:Transformation
Returns all transitive predecessorTransformation
s of thisTransformation
. This is, for example, used when determining whether a feedback edge of an iteration actually has the iteration head as a predecessor.- Specified by:
getTransitivePredecessorsInternal
in classTransformation<F>
- Returns:
- The list of transitive predecessors.
-
getInputs
public List<Transformation<?>> getInputs()
Description copied from class:Transformation
Returns thetransformations
that are the immediate predecessors of the current transformation in the transformation graph.- Specified by:
getInputs
in classTransformation<F>
-
-