Class ForwardForConsecutiveHashPartitioner<T>
- Type Parameters:
T- Type of the elements in the Stream
- All Implemented Interfaces:
Serializable,ChannelSelector<SerializationDelegate<StreamRecord<T>>>
A --[hash]--> B --[hash]--> C
|
V
A --[hash]--> B --[forward]--> C
However, sometimes the consecutive hash operators are not chained (e.g. multiple inputs), and this kind of forward partitioners will turn into forward job edges. These forward edges still have the consecutive hash assumption, so that they cannot be changed into rescale/rebalance edges, otherwise it can lead to incorrect results. This prevents the adaptive batch scheduler from determining parallelism for other forward edge downstream job vertices(see FLINK-25046).
To solve it, we introduce the ForwardForConsecutiveHashPartitioner. When SQL planner
optimizes the case of multiple consecutive and the same hash shuffles, it should use this
partitioner, and then the runtime framework will change it to forward/hash after the operator
chain creation.
A --[hash]--> B --[hash]--> C
|
V
A --[hash]--> B --[ForwardForConsecutiveHash]--> C
This partitioner will be converted to following partitioners after the operator chain creation:
1. Be converted to ForwardPartitioner if this partitioner is intra-chain.
2. Be converted to hashPartitioner if this
partitioner is inter-chain.
This partitioner should only be used for SQL Batch jobs and when using AdaptiveBatchScheduler.
- See Also:
-
Field Summary
Fields inherited from class org.apache.flink.streaming.runtime.partitioner.StreamPartitioner
numberOfChannels -
Constructor Summary
ConstructorsConstructorDescriptionForwardForConsecutiveHashPartitioner(StreamPartitioner<T> hashPartitioner) Create a new ForwardForConsecutiveHashPartitioner. -
Method Summary
Modifier and TypeMethodDescriptioncopy()voidDefines the behavior of this partitioner, when downstream rescaled during recovery of in-flight data.booleanintselectChannel(SerializationDelegate<StreamRecord<T>> record) Returns the logical channel index, to which the given record should be written.Methods inherited from class org.apache.flink.streaming.runtime.partitioner.ForwardPartitioner
getUpstreamSubtaskStateMapper, toStringMethods inherited from class org.apache.flink.streaming.runtime.partitioner.StreamPartitioner
equals, hashCode, isBroadcast, isSupportsUnalignedCheckpoint, setup
-
Constructor Details
-
ForwardForConsecutiveHashPartitioner
Create a new ForwardForConsecutiveHashPartitioner.- Parameters:
hashPartitioner- the HashPartitioner
-
-
Method Details
-
copy
- Overrides:
copyin classForwardPartitioner<T>
-
getDownstreamSubtaskStateMapper
Description copied from class:StreamPartitionerDefines the behavior of this partitioner, when downstream rescaled during recovery of in-flight data.- Overrides:
getDownstreamSubtaskStateMapperin classForwardPartitioner<T>
-
isPointwise
public boolean isPointwise()- Overrides:
isPointwisein classForwardPartitioner<T>
-
disableUnalignedCheckpoints
public void disableUnalignedCheckpoints()- Overrides:
disableUnalignedCheckpointsin classStreamPartitioner<T>
-
selectChannel
Description copied from interface:ChannelSelectorReturns the logical channel index, to which the given record should be written. It is illegal to call this method for broadcast channel selectors and this method can remain not implemented in that case (for example by throwingUnsupportedOperationException).- Specified by:
selectChannelin interfaceChannelSelector<T>- Overrides:
selectChannelin classForwardPartitioner<T>- Parameters:
record- the record to determine the output channels for.- Returns:
- an integer number which indicates the index of the output channel through which the record shall be forwarded.
-
getHashPartitioner
-