Class ResultPartition
- java.lang.Object
-
- org.apache.flink.runtime.io.network.partition.ResultPartition
-
- All Implemented Interfaces:
AutoCloseable
,AvailabilityProvider
,ResultPartitionWriter
- Direct Known Subclasses:
BufferWritingResultPartition
,SortMergeResultPartition
,TieredResultPartition
public abstract class ResultPartition extends Object implements ResultPartitionWriter
A result partition for data produced by a single task.This class is the runtime part of a logical
IntermediateResultPartition
. Essentially, a result partition is a collection ofBuffer
instances. The buffers are organized in one or moreResultSubpartition
instances or in a joint structure which further partition the data depending on the number of consuming tasks and the dataDistributionPattern
.Tasks, which consume a result partition have to request one of its subpartitions. The request happens either remotely (see
RemoteInputChannel
) or locally (seeLocalInputChannel
)Life-cycle
The life-cycle of each result partition has three (possibly overlapping) phases:
- Produce:
- Consume:
- Release:
Buffer management
State management
-
-
Nested Class Summary
-
Nested classes/interfaces inherited from interface org.apache.flink.runtime.io.AvailabilityProvider
AvailabilityProvider.AvailabilityHelper
-
-
Field Summary
Fields Modifier and Type Field Description protected BufferCompressor
bufferCompressor
Used to compress buffer to reduce IO.protected BufferPool
bufferPool
protected static org.slf4j.Logger
LOG
protected Counter
numBuffersOut
protected Counter
numBytesOut
protected int
numSubpartitions
protected ResultPartitionID
partitionId
protected ResultPartitionManager
partitionManager
protected ResultPartitionType
partitionType
Type of this partition.protected ResultPartitionBytesCounter
resultPartitionBytes
-
Fields inherited from interface org.apache.flink.runtime.io.AvailabilityProvider
AVAILABLE
-
-
Constructor Summary
Constructors Constructor Description ResultPartition(String owningTaskName, int partitionIndex, ResultPartitionID partitionId, ResultPartitionType partitionType, int numSubpartitions, int numTargetKeyGroups, ResultPartitionManager partitionManager, BufferCompressor bufferCompressor, SupplierWithException<BufferPool,IOException> bufferPoolFactory)
-
Method Summary
All Methods Instance Methods Abstract Methods Concrete Methods Modifier and Type Method Description protected boolean
canBeCompressed(Buffer buffer)
Whether the buffer can be compressed or not.protected void
checkInProduceState()
void
close()
Closes the partition writer which releases the allocated resource, for example the buffer pool.protected abstract ResultSubpartitionView
createSubpartitionView(int index, BufferAvailabilityListener availabilityListener)
Returns a reader for the subpartition with the given index.ResultSubpartitionView
createSubpartitionView(ResultSubpartitionIndexSet indexSet, BufferAvailabilityListener availabilityListener)
Returns a reader for the subpartition with the given index range.void
fail(Throwable throwable)
Fail the production of the partition.void
finish()
Finishes the result partition.CompletableFuture<Void>
getAllDataProcessedFuture()
Gets the future indicating whether all the records has been processed by the downstream tasks.CompletableFuture<?>
getAvailableFuture()
BufferPool
getBufferPool()
Throwable
getFailureCause()
abstract int
getNumberOfQueuedBuffers()
Returns the total number of queued buffers of all subpartitions.abstract int
getNumberOfQueuedBuffers(int targetSubpartition)
Returns the number of queued buffers of the given target subpartition.int
getNumberOfSubpartitions()
int
getNumTargetKeyGroups()
String
getOwningTaskName()
ResultPartitionID
getPartitionId()
int
getPartitionIndex()
ResultPartitionManager
getPartitionManager()
ResultPartitionType
getPartitionType()
Returns the type of this result partition.ResultPartitionBytesCounter
getResultPartitionBytes()
abstract long
getSizeOfQueuedBuffersUnsafe()
Returns the total size in bytes of queued buffers of all subpartitions.boolean
isFinished()
boolean
isNumberOfPartitionConsumerUndefined()
void
isNumberOfPartitionConsumerUndefined(boolean isNumberOfPartitionConsumerUndefined)
boolean
isReleased()
Whether this partition is released.void
notifyEndOfData(StopMode mode)
Notifies the downstream tasks that thisResultPartitionWriter
have emitted all the user records.void
onSubpartitionAllDataProcessed(int subpartition)
The subpartition notifies that the corresponding downstream task have processed all the user records.void
release()
void
release(Throwable cause)
Releases the partition writer which releases the produced data and no reader can consume the partition any more.protected abstract void
releaseInternal()
Releases all produced data including both those stored in memory and persisted on disk.void
setMaxOverdraftBuffersPerGate(int maxOverdraftBuffersPerGate)
Sets the max overdraft buffer size of per gate.void
setMetricGroup(TaskIOMetricGroup metrics)
Sets the metric group for theResultPartitionWriter
.void
setup()
Registers a buffer pool with this result partition.protected abstract void
setupInternal()
Do the subclass's own setup operation.String
toString()
-
Methods inherited from class java.lang.Object
clone, equals, finalize, getClass, hashCode, notify, notifyAll, wait, wait, wait
-
Methods inherited from interface org.apache.flink.runtime.io.AvailabilityProvider
isApproximatelyAvailable, isAvailable
-
Methods inherited from interface org.apache.flink.runtime.io.network.api.writer.ResultPartitionWriter
abortCheckpoint, alignedBarrierTimeout, broadcastEvent, broadcastRecord, emitRecord, flush, flushAll
-
-
-
-
Field Detail
-
LOG
protected static final org.slf4j.Logger LOG
-
partitionId
protected final ResultPartitionID partitionId
-
partitionType
protected final ResultPartitionType partitionType
Type of this partition. Defines the concrete subpartition implementation to use.
-
partitionManager
protected final ResultPartitionManager partitionManager
-
numSubpartitions
protected final int numSubpartitions
-
bufferPool
protected BufferPool bufferPool
-
bufferCompressor
@Nullable protected final BufferCompressor bufferCompressor
Used to compress buffer to reduce IO.
-
numBytesOut
protected Counter numBytesOut
-
numBuffersOut
protected Counter numBuffersOut
-
resultPartitionBytes
protected ResultPartitionBytesCounter resultPartitionBytes
-
-
Constructor Detail
-
ResultPartition
public ResultPartition(String owningTaskName, int partitionIndex, ResultPartitionID partitionId, ResultPartitionType partitionType, int numSubpartitions, int numTargetKeyGroups, ResultPartitionManager partitionManager, @Nullable BufferCompressor bufferCompressor, SupplierWithException<BufferPool,IOException> bufferPoolFactory)
-
-
Method Detail
-
setup
public void setup() throws IOException
Registers a buffer pool with this result partition.There is one pool for each result partition, which is shared by all its sub partitions.
The pool is registered with the partition *after* it as been constructed in order to conform to the life-cycle of task registrations in the
TaskExecutor
.- Specified by:
setup
in interfaceResultPartitionWriter
- Throws:
IOException
-
setupInternal
protected abstract void setupInternal() throws IOException
Do the subclass's own setup operation.- Throws:
IOException
-
getOwningTaskName
public String getOwningTaskName()
-
getPartitionId
public ResultPartitionID getPartitionId()
- Specified by:
getPartitionId
in interfaceResultPartitionWriter
-
getPartitionIndex
public int getPartitionIndex()
-
getNumberOfSubpartitions
public int getNumberOfSubpartitions()
- Specified by:
getNumberOfSubpartitions
in interfaceResultPartitionWriter
-
getBufferPool
public BufferPool getBufferPool()
-
isNumberOfPartitionConsumerUndefined
public void isNumberOfPartitionConsumerUndefined(boolean isNumberOfPartitionConsumerUndefined)
-
isNumberOfPartitionConsumerUndefined
public boolean isNumberOfPartitionConsumerUndefined()
-
getNumberOfQueuedBuffers
public abstract int getNumberOfQueuedBuffers()
Returns the total number of queued buffers of all subpartitions.
-
getSizeOfQueuedBuffersUnsafe
public abstract long getSizeOfQueuedBuffersUnsafe()
Returns the total size in bytes of queued buffers of all subpartitions.
-
getNumberOfQueuedBuffers
public abstract int getNumberOfQueuedBuffers(int targetSubpartition)
Returns the number of queued buffers of the given target subpartition.
-
setMaxOverdraftBuffersPerGate
public void setMaxOverdraftBuffersPerGate(int maxOverdraftBuffersPerGate)
Description copied from interface:ResultPartitionWriter
Sets the max overdraft buffer size of per gate.- Specified by:
setMaxOverdraftBuffersPerGate
in interfaceResultPartitionWriter
-
getPartitionType
public ResultPartitionType getPartitionType()
Returns the type of this result partition.- Returns:
- result partition type
-
getResultPartitionBytes
public ResultPartitionBytesCounter getResultPartitionBytes()
-
notifyEndOfData
public void notifyEndOfData(StopMode mode) throws IOException
Description copied from interface:ResultPartitionWriter
Notifies the downstream tasks that thisResultPartitionWriter
have emitted all the user records.- Specified by:
notifyEndOfData
in interfaceResultPartitionWriter
- Parameters:
mode
- tells if we should flush all records or not (it is false in case of stop-with-savepoint (--no-drain))- Throws:
IOException
-
getAllDataProcessedFuture
public CompletableFuture<Void> getAllDataProcessedFuture()
Description copied from interface:ResultPartitionWriter
Gets the future indicating whether all the records has been processed by the downstream tasks.- Specified by:
getAllDataProcessedFuture
in interfaceResultPartitionWriter
-
onSubpartitionAllDataProcessed
public void onSubpartitionAllDataProcessed(int subpartition)
The subpartition notifies that the corresponding downstream task have processed all the user records.- Parameters:
subpartition
- The index of the subpartition sending the notification.- See Also:
EndOfData
-
finish
public void finish() throws IOException
Finishes the result partition.After this operation, it is not possible to add further data to the result partition.
For BLOCKING results, this will trigger the deployment of consuming tasks.
- Specified by:
finish
in interfaceResultPartitionWriter
- Throws:
IOException
-
isFinished
public boolean isFinished()
- Specified by:
isFinished
in interfaceResultPartitionWriter
-
release
public void release()
-
release
public void release(Throwable cause)
Description copied from interface:ResultPartitionWriter
Releases the partition writer which releases the produced data and no reader can consume the partition any more.- Specified by:
release
in interfaceResultPartitionWriter
-
releaseInternal
protected abstract void releaseInternal()
Releases all produced data including both those stored in memory and persisted on disk.
-
close
public void close()
Description copied from interface:ResultPartitionWriter
Closes the partition writer which releases the allocated resource, for example the buffer pool.- Specified by:
close
in interfaceAutoCloseable
- Specified by:
close
in interfaceResultPartitionWriter
-
fail
public void fail(@Nullable Throwable throwable)
Description copied from interface:ResultPartitionWriter
Fail the production of the partition.This method propagates non-
null
failure causes to consumers on a best-effort basis. This call also leads to the release of all resources associated with the partition. Closing of the partition is still needed afterwards if it has not been done before.- Specified by:
fail
in interfaceResultPartitionWriter
- Parameters:
throwable
- failure cause
-
getFailureCause
public Throwable getFailureCause()
-
getNumTargetKeyGroups
public int getNumTargetKeyGroups()
- Specified by:
getNumTargetKeyGroups
in interfaceResultPartitionWriter
-
setMetricGroup
public void setMetricGroup(TaskIOMetricGroup metrics)
Description copied from interface:ResultPartitionWriter
Sets the metric group for theResultPartitionWriter
.- Specified by:
setMetricGroup
in interfaceResultPartitionWriter
-
createSubpartitionView
public ResultSubpartitionView createSubpartitionView(ResultSubpartitionIndexSet indexSet, BufferAvailabilityListener availabilityListener) throws IOException
Description copied from interface:ResultPartitionWriter
Returns a reader for the subpartition with the given index range.- Specified by:
createSubpartitionView
in interfaceResultPartitionWriter
- Throws:
IOException
-
createSubpartitionView
protected abstract ResultSubpartitionView createSubpartitionView(int index, BufferAvailabilityListener availabilityListener) throws IOException
Returns a reader for the subpartition with the given index.Given that the function to merge outputs from multiple subpartition views is supported uniformly in
UnionResultSubpartitionView
, subclasses ofResultPartition
only needs to take care of creating subpartition view for a single subpartition.- Throws:
IOException
-
isReleased
public boolean isReleased()
Whether this partition is released.A partition is released when each subpartition is either consumed and communication is closed by consumer or failed. A partition is also released if task is cancelled.
- Specified by:
isReleased
in interfaceResultPartitionWriter
-
getAvailableFuture
public CompletableFuture<?> getAvailableFuture()
- Specified by:
getAvailableFuture
in interfaceAvailabilityProvider
- Returns:
- a future that is completed if the respective provider is available.
-
checkInProduceState
protected void checkInProduceState() throws IllegalStateException
- Throws:
IllegalStateException
-
getPartitionManager
@VisibleForTesting public ResultPartitionManager getPartitionManager()
-
canBeCompressed
protected boolean canBeCompressed(Buffer buffer)
Whether the buffer can be compressed or not. Note that event is not compressed because it is usually small and the size can become even larger after compression.
-
-