Enum ResultPartitionType
- java.lang.Object
-
- java.lang.Enum<ResultPartitionType>
-
- org.apache.flink.runtime.io.network.partition.ResultPartitionType
-
- All Implemented Interfaces:
Serializable
,Comparable<ResultPartitionType>
public enum ResultPartitionType extends Enum<ResultPartitionType>
Type of a result partition.
-
-
Enum Constant Summary
Enum Constants Enum Constant Description BLOCKING
Blocking partitions represent blocking data exchanges, where the data stream is first fully produced and then consumed.BLOCKING_PERSISTENT
BLOCKING_PERSISTENT partitions are similar toBLOCKING
partitions, but have a user-specified life cycle.HYBRID_FULL
Hybrid partitions with a bounded (local) buffer pool to support downstream task to simultaneous reading and writing shuffle data.HYBRID_SELECTIVE
HYBRID_SELECTIVE partitions are similar toHYBRID_FULL
partitions, but it is not re-consumable.PIPELINED
A pipelined streaming data exchange.PIPELINED_APPROXIMATE
Pipelined partitions with a bounded (local) buffer pool to support downstream task to continue consuming data after reconnection in Approximate Local-Recovery.PIPELINED_BOUNDED
Pipelined partitions with a bounded (local) buffer pool.
-
Method Summary
All Methods Static Methods Instance Methods Concrete Methods Modifier and Type Method Description boolean
canBePipelinedConsumed()
return if this partition's upstream and downstream support scheduling in the same time.boolean
isBlockingOrBlockingPersistentResultPartition()
isBlockingOrBlockingPersistentResultPartition()
is used to judge whether it is the specifiedBLOCKING
orBLOCKING_PERSISTENT
resultPartitionType.boolean
isBounded()
Whether this partition uses a limited number of (network) buffers or not.boolean
isHybridResultPartition()
isHybridResultPartition()
is used to judge whether it is the specifiedHYBRID_FULL
orHYBRID_SELECTIVE
resultPartitionType.boolean
isPersistent()
boolean
isPipelinedOrPipelinedBoundedResultPartition()
isPipelinedOrPipelinedBoundedResultPartition()
is used to judge whether it is the specifiedPIPELINED
orPIPELINED_BOUNDED
resultPartitionType.boolean
isReconsumable()
boolean
isReleaseByScheduler()
boolean
isReleaseByUpstream()
boolean
mustBePipelinedConsumed()
return if this partition's upstream and downstream must be scheduled in the same time.boolean
supportCompression()
static ResultPartitionType
valueOf(String name)
Returns the enum constant of this type with the specified name.static ResultPartitionType[]
values()
Returns an array containing the constants of this enum type, in the order they are declared.
-
-
-
Enum Constant Detail
-
BLOCKING
public static final ResultPartitionType BLOCKING
Blocking partitions represent blocking data exchanges, where the data stream is first fully produced and then consumed. This is an option that is only applicable to bounded streams and can be used in bounded stream runtime and recovery.Blocking partitions can be consumed multiple times and concurrently.
The partition is not automatically released after being consumed (like for example the
PIPELINED
partitions), but only released through the scheduler, when it determines that the partition is no longer needed.
-
BLOCKING_PERSISTENT
public static final ResultPartitionType BLOCKING_PERSISTENT
BLOCKING_PERSISTENT partitions are similar toBLOCKING
partitions, but have a user-specified life cycle.BLOCKING_PERSISTENT partitions are dropped upon explicit API calls to the JobManager or ResourceManager, rather than by the scheduler.
Otherwise, the partition may only be dropped by safety-nets during failure handling scenarios, like when the TaskManager exits or when the TaskManager loses connection to JobManager / ResourceManager for too long.
-
PIPELINED
public static final ResultPartitionType PIPELINED
A pipelined streaming data exchange. This is applicable to both bounded and unbounded streams.Pipelined results can be consumed only once by a single consumer and are automatically disposed when the stream has been consumed.
This result partition type may keep an arbitrary amount of data in-flight, in contrast to the
PIPELINED_BOUNDED
variant.
-
PIPELINED_BOUNDED
public static final ResultPartitionType PIPELINED_BOUNDED
Pipelined partitions with a bounded (local) buffer pool.For streaming jobs, a fixed limit on the buffer pool size should help avoid that too much data is being buffered and checkpoint barriers are delayed. In contrast to limiting the overall network buffer pool size, this, however, still allows to be flexible with regards to the total number of partitions by selecting an appropriately big network buffer pool size.
For batch jobs, it will be best to keep this unlimited (
PIPELINED
) since there are no checkpoint barriers.
-
PIPELINED_APPROXIMATE
public static final ResultPartitionType PIPELINED_APPROXIMATE
Pipelined partitions with a bounded (local) buffer pool to support downstream task to continue consuming data after reconnection in Approximate Local-Recovery.Pipelined results can be consumed only once by a single consumer at one time.
PIPELINED_APPROXIMATE
is different fromPIPELINED
andPIPELINED_BOUNDED
in thatPIPELINED_APPROXIMATE
partition can be reconnected after down stream task fails.
-
HYBRID_FULL
public static final ResultPartitionType HYBRID_FULL
Hybrid partitions with a bounded (local) buffer pool to support downstream task to simultaneous reading and writing shuffle data.Hybrid partitions can be consumed any time, whether fully produced or not.
HYBRID_FULL partitions is re-consumable, so double calculation can be avoided during failover.
-
HYBRID_SELECTIVE
public static final ResultPartitionType HYBRID_SELECTIVE
HYBRID_SELECTIVE partitions are similar toHYBRID_FULL
partitions, but it is not re-consumable.
-
-
Method Detail
-
values
public static ResultPartitionType[] values()
Returns an array containing the constants of this enum type, in the order they are declared. This method may be used to iterate over the constants as follows:for (ResultPartitionType c : ResultPartitionType.values()) System.out.println(c);
- Returns:
- an array containing the constants of this enum type, in the order they are declared
-
valueOf
public static ResultPartitionType valueOf(String name)
Returns the enum constant of this type with the specified name. The string must match exactly an identifier used to declare an enum constant in this type. (Extraneous whitespace characters are not permitted.)- Parameters:
name
- the name of the enum constant to be returned.- Returns:
- the enum constant with the specified name
- Throws:
IllegalArgumentException
- if this enum type has no constant with the specified nameNullPointerException
- if the argument is null
-
mustBePipelinedConsumed
public boolean mustBePipelinedConsumed()
return if this partition's upstream and downstream must be scheduled in the same time.
-
canBePipelinedConsumed
public boolean canBePipelinedConsumed()
return if this partition's upstream and downstream support scheduling in the same time.
-
isReleaseByScheduler
public boolean isReleaseByScheduler()
-
isReleaseByUpstream
public boolean isReleaseByUpstream()
-
isBlockingOrBlockingPersistentResultPartition
public boolean isBlockingOrBlockingPersistentResultPartition()
isBlockingOrBlockingPersistentResultPartition()
is used to judge whether it is the specifiedBLOCKING
orBLOCKING_PERSISTENT
resultPartitionType.this method suitable for judgment conditions related to the specific implementation of
ResultPartitionType
.this method not related to data consumption and partition release. As for the logic related to partition release, use
isReleaseByScheduler()
instead, and as consume type, usemustBePipelinedConsumed()
orcanBePipelinedConsumed()
instead.
-
isHybridResultPartition
public boolean isHybridResultPartition()
isHybridResultPartition()
is used to judge whether it is the specifiedHYBRID_FULL
orHYBRID_SELECTIVE
resultPartitionType.this method suitable for judgment conditions related to the specific implementation of
ResultPartitionType
.this method not related to data consumption and partition release. As for the logic related to partition release, use
isReleaseByScheduler()
instead, and as consume type, usemustBePipelinedConsumed()
orcanBePipelinedConsumed()
instead.
-
isPipelinedOrPipelinedBoundedResultPartition
public boolean isPipelinedOrPipelinedBoundedResultPartition()
isPipelinedOrPipelinedBoundedResultPartition()
is used to judge whether it is the specifiedPIPELINED
orPIPELINED_BOUNDED
resultPartitionType.This method suitable for judgment conditions related to the specific implementation of
ResultPartitionType
.This method not related to data consumption and partition release. As for the logic related to partition release, use
isReleaseByScheduler()
instead, and as consume type, usemustBePipelinedConsumed()
orcanBePipelinedConsumed()
instead.
-
isBounded
public boolean isBounded()
Whether this partition uses a limited number of (network) buffers or not.- Returns:
- true if the number of buffers should be bound to some limit
-
isPersistent
public boolean isPersistent()
-
supportCompression
public boolean supportCompression()
-
isReconsumable
public boolean isReconsumable()
-
-