Class RescalePartitioner<T>
- java.lang.Object
-
- org.apache.flink.streaming.runtime.partitioner.StreamPartitioner<T>
-
- org.apache.flink.streaming.runtime.partitioner.RescalePartitioner<T>
-
- Type Parameters:
T
- Type of the elements in the Stream being rescaled
- All Implemented Interfaces:
Serializable
,ChannelSelector<SerializationDelegate<StreamRecord<T>>>
@Internal public class RescalePartitioner<T> extends StreamPartitioner<T>
Partitioner that distributes the data equally by cycling through the output channels. This distributes only to a subset of downstream nodes becauseStreamingJobGraphGenerator
instantiates aDistributionPattern.POINTWISE
distribution pattern when encounteringRescalePartitioner
.The subset of downstream operations to which the upstream operation sends elements depends on the degree of parallelism of both the upstream and downstream operation. For example, if the upstream operation has parallelism 2 and the downstream operation has parallelism 4, then one upstream operation would distribute elements to two downstream operations while the other upstream operation would distribute to the other two downstream operations. If, on the other hand, the downstream operation has parallelism 2 while the upstream operation has parallelism 4 then two upstream operations will distribute to one downstream operation while the other two upstream operations will distribute to the other downstream operations.
In cases where the different parallelisms are not multiples of each other one or several downstream operations will have a differing number of inputs from upstream operations.
- See Also:
- Serialized Form
-
-
Field Summary
-
Fields inherited from class org.apache.flink.streaming.runtime.partitioner.StreamPartitioner
numberOfChannels
-
-
Constructor Summary
Constructors Constructor Description RescalePartitioner()
-
Method Summary
All Methods Instance Methods Concrete Methods Modifier and Type Method Description StreamPartitioner<T>
copy()
SubtaskStateMapper
getDownstreamSubtaskStateMapper()
Defines the behavior of this partitioner, when downstream rescaled during recovery of in-flight data.SubtaskStateMapper
getUpstreamSubtaskStateMapper()
Defines the behavior of this partitioner, when upstream rescaled during recovery of in-flight data.boolean
isPointwise()
int
selectChannel(SerializationDelegate<StreamRecord<T>> record)
Returns the logical channel index, to which the given record should be written.String
toString()
-
Methods inherited from class org.apache.flink.streaming.runtime.partitioner.StreamPartitioner
disableUnalignedCheckpoints, equals, hashCode, isBroadcast, isSupportsUnalignedCheckpoint, setup
-
-
-
-
Method Detail
-
selectChannel
public int selectChannel(SerializationDelegate<StreamRecord<T>> record)
Description copied from interface:ChannelSelector
Returns 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
).- 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.
-
getDownstreamSubtaskStateMapper
public SubtaskStateMapper getDownstreamSubtaskStateMapper()
Description copied from class:StreamPartitioner
Defines the behavior of this partitioner, when downstream rescaled during recovery of in-flight data.- Specified by:
getDownstreamSubtaskStateMapper
in classStreamPartitioner<T>
-
getUpstreamSubtaskStateMapper
public SubtaskStateMapper getUpstreamSubtaskStateMapper()
Description copied from class:StreamPartitioner
Defines the behavior of this partitioner, when upstream rescaled during recovery of in-flight data.- Overrides:
getUpstreamSubtaskStateMapper
in classStreamPartitioner<T>
-
copy
public StreamPartitioner<T> copy()
- Specified by:
copy
in classStreamPartitioner<T>
-
isPointwise
public boolean isPointwise()
- Specified by:
isPointwise
in classStreamPartitioner<T>
-
-