Package org.apache.flink.runtime.state
Class KeyGroupRangeAssignment
- java.lang.Object
-
- org.apache.flink.runtime.state.KeyGroupRangeAssignment
-
public final class KeyGroupRangeAssignment extends Object
-
-
Field Summary
Fields Modifier and Type Field Description static int
DEFAULT_LOWER_BOUND_MAX_PARALLELISM
The default lower bound for max parallelism if nothing was configured by the user.static int
UPPER_BOUND_MAX_PARALLELISM
The (inclusive) upper bound for max parallelism.
-
Method Summary
All Methods Static Methods Concrete Methods Modifier and Type Method Description static int
assignKeyToParallelOperator(Object key, int maxParallelism, int parallelism)
Assigns the given key to a parallel operator index.static int
assignToKeyGroup(Object key, int maxParallelism)
Assigns the given key to a key-group index.static void
checkParallelismPreconditions(int parallelism)
static int
computeDefaultMaxParallelism(int operatorParallelism)
Computes a default maximum parallelism from the operator parallelism.static int
computeKeyGroupForKeyHash(int keyHash, int maxParallelism)
Assigns the given key to a key-group index.static KeyGroupRange
computeKeyGroupRangeForOperatorIndex(int maxParallelism, int parallelism, int operatorIndex)
Computes the range of key-groups that are assigned to a given operator under the given parallelism and maximum parallelism.static int
computeOperatorIndexForKeyGroup(int maxParallelism, int parallelism, int keyGroupId)
Computes the index of the operator to which a key-group belongs under the given parallelism and maximum parallelism.
-
-
-
Field Detail
-
DEFAULT_LOWER_BOUND_MAX_PARALLELISM
public static final int DEFAULT_LOWER_BOUND_MAX_PARALLELISM
The default lower bound for max parallelism if nothing was configured by the user. We have this to allow users some degree of scale-up in case they forgot to configure maximum parallelism explicitly.- See Also:
- Constant Field Values
-
UPPER_BOUND_MAX_PARALLELISM
public static final int UPPER_BOUND_MAX_PARALLELISM
The (inclusive) upper bound for max parallelism.- See Also:
- Constant Field Values
-
-
Method Detail
-
assignKeyToParallelOperator
public static int assignKeyToParallelOperator(Object key, int maxParallelism, int parallelism)
Assigns the given key to a parallel operator index.- Parameters:
key
- the key to assignmaxParallelism
- the maximum supported parallelism, aka the number of key-groups.parallelism
- the current parallelism of the operator- Returns:
- the index of the parallel operator to which the given key should be routed.
-
assignToKeyGroup
public static int assignToKeyGroup(Object key, int maxParallelism)
Assigns the given key to a key-group index.- Parameters:
key
- the key to assignmaxParallelism
- the maximum supported parallelism, aka the number of key-groups.- Returns:
- the key-group to which the given key is assigned
-
computeKeyGroupForKeyHash
public static int computeKeyGroupForKeyHash(int keyHash, int maxParallelism)
Assigns the given key to a key-group index.- Parameters:
keyHash
- the hash of the key to assignmaxParallelism
- the maximum supported parallelism, aka the number of key-groups.- Returns:
- the key-group to which the given key is assigned
-
computeKeyGroupRangeForOperatorIndex
public static KeyGroupRange computeKeyGroupRangeForOperatorIndex(int maxParallelism, int parallelism, int operatorIndex)
Computes the range of key-groups that are assigned to a given operator under the given parallelism and maximum parallelism.IMPORTANT: maxParallelism must be <= Short.MAX_VALUE + 1 to avoid rounding problems in this method. If we ever want to go beyond this boundary, this method must perform arithmetic on long values.
- Parameters:
maxParallelism
- Maximal parallelism that the job was initially created with.parallelism
- The current parallelism under which the job runs. Must be <= maxParallelism.operatorIndex
- index of a operatorIndex. 0 <= operatorIndex < parallelism.- Returns:
- the computed key-group range for the operator.
-
computeOperatorIndexForKeyGroup
public static int computeOperatorIndexForKeyGroup(int maxParallelism, int parallelism, int keyGroupId)
Computes the index of the operator to which a key-group belongs under the given parallelism and maximum parallelism.IMPORTANT: maxParallelism must be <= Short.MAX_VALUE + 1 to avoid rounding problems in this method. If we ever want to go beyond this boundary, this method must perform arithmetic on long values.
- Parameters:
maxParallelism
- Maximal parallelism that the job was initially created with. 0 < parallelism <= maxParallelism <= Short.MAX_VALUE + 1 must hold.parallelism
- The current parallelism under which the job runs. Must be <= maxParallelism.keyGroupId
- Id of a key-group. 0 <= keyGroupID < maxParallelism.- Returns:
- The index of the operator to which elements from the given key-group should be routed under the given parallelism and maxParallelism.
-
computeDefaultMaxParallelism
public static int computeDefaultMaxParallelism(int operatorParallelism)
Computes a default maximum parallelism from the operator parallelism. This is used in case the user has not explicitly configured a maximum parallelism to still allow a certain degree of scale-up.- Parameters:
operatorParallelism
- the operator parallelism as basis for computation.- Returns:
- the computed default maximum parallelism.
-
checkParallelismPreconditions
public static void checkParallelismPreconditions(int parallelism)
-
-