Class FeedbackTransformation<T>
- java.lang.Object
-
- org.apache.flink.api.dag.Transformation<T>
-
- org.apache.flink.streaming.api.transformations.FeedbackTransformation<T>
-
- Type Parameters:
T
- The type of the input elements and the feedback elements.
@Internal public class FeedbackTransformation<T> extends Transformation<T>
This represents a feedback point in a topology.This is different from how iterations work in batch processing. Once a feedback point is defined you can connect one or several
Transformations
as a feedback edges. Operations downstream from the feedback point will receive elements from the input of this feedback point and from the feedback edges.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 type of the input
Transformation
and the feedbackTransformation
must match.
-
-
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 FeedbackTransformation(Transformation<T> input, Long waitTime)
Creates a newFeedbackTransformation
from the given input.
-
Method Summary
All Methods Instance Methods Concrete Methods Modifier and Type Method Description void
addFeedbackEdge(Transformation<T> transform)
Adds a feedback edge.List<Transformation<T>>
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
-
FeedbackTransformation
public FeedbackTransformation(Transformation<T> input, Long waitTime)
Creates a newFeedbackTransformation
from the given input.- Parameters:
input
- The inputTransformation
waitTime
- 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<T> transform)
Adds a feedback edge. The parallelism of theTransformation
must match the parallelism of the inputTransformation
of thisFeedbackTransformation
- Parameters:
transform
- The new feedbackTransformation
.
-
getFeedbackEdges
public List<Transformation<T>> 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<T>
- 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<T>
-
-