public abstract class ResultPartition extends Object implements ResultPartitionWriter
This class is the runtime part of a logical IntermediateResultPartition
. Essentially,
a result partition is a collection of Buffer
instances. The buffers are organized in one
or more ResultSubpartition
instances or in a joint structure which further partition the
data depending on the number of consuming tasks and the data DistributionPattern
.
Tasks, which consume a result partition have to request one of its subpartitions. The request
happens either remotely (see RemoteInputChannel
) or locally (see LocalInputChannel
)
The life-cycle of each result partition has three (possibly overlapping) phases:
AvailabilityProvider.AvailabilityHelper
Modifier and Type | Field and 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 |
AVAILABLE
Constructor and Description |
---|
ResultPartition(String owningTaskName,
int partitionIndex,
ResultPartitionID partitionId,
ResultPartitionType partitionType,
int numSubpartitions,
int numTargetKeyGroups,
ResultPartitionManager partitionManager,
BufferCompressor bufferCompressor,
SupplierWithException<BufferPool,IOException> bufferPoolFactory) |
Modifier and Type | Method and 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.
|
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.
|
abstract long |
getSizeOfQueuedBuffersUnsafe()
Returns the total size in bytes of queued buffers of all subpartitions.
|
boolean |
isFinished() |
boolean |
isReleased()
Whether this partition is released.
|
void |
notifyEndOfData(StopMode mode)
Notifies the downstream tasks that this
ResultPartitionWriter 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 the
ResultPartitionWriter . |
void |
setup()
Registers a buffer pool with this result partition.
|
protected abstract void |
setupInternal()
Do the subclass's own setup operation.
|
String |
toString() |
clone, equals, finalize, getClass, hashCode, notify, notifyAll, wait, wait, wait
abortCheckpoint, alignedBarrierTimeout, broadcastEvent, broadcastRecord, createSubpartitionView, emitRecord, flush, flushAll
and, isApproximatelyAvailable, isAvailable, or
protected static final org.slf4j.Logger LOG
protected final ResultPartitionID partitionId
protected final ResultPartitionType partitionType
protected final ResultPartitionManager partitionManager
protected final int numSubpartitions
protected BufferPool bufferPool
@Nullable protected final BufferCompressor bufferCompressor
protected Counter numBytesOut
protected Counter numBuffersOut
protected ResultPartitionBytesCounter resultPartitionBytes
public ResultPartition(String owningTaskName, int partitionIndex, ResultPartitionID partitionId, ResultPartitionType partitionType, int numSubpartitions, int numTargetKeyGroups, ResultPartitionManager partitionManager, @Nullable BufferCompressor bufferCompressor, SupplierWithException<BufferPool,IOException> bufferPoolFactory)
public void setup() throws IOException
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
.
setup
in interface ResultPartitionWriter
IOException
protected abstract void setupInternal() throws IOException
IOException
public String getOwningTaskName()
public ResultPartitionID getPartitionId()
getPartitionId
in interface ResultPartitionWriter
public int getPartitionIndex()
public int getNumberOfSubpartitions()
getNumberOfSubpartitions
in interface ResultPartitionWriter
public BufferPool getBufferPool()
public abstract int getNumberOfQueuedBuffers()
public abstract long getSizeOfQueuedBuffersUnsafe()
public abstract int getNumberOfQueuedBuffers(int targetSubpartition)
public void setMaxOverdraftBuffersPerGate(int maxOverdraftBuffersPerGate)
ResultPartitionWriter
setMaxOverdraftBuffersPerGate
in interface ResultPartitionWriter
public ResultPartitionType getPartitionType()
public void notifyEndOfData(StopMode mode) throws IOException
ResultPartitionWriter
ResultPartitionWriter
have emitted all the
user records.notifyEndOfData
in interface ResultPartitionWriter
mode
- tells if we should flush all records or not (it is false in case of
stop-with-savepoint (--no-drain))IOException
public CompletableFuture<Void> getAllDataProcessedFuture()
ResultPartitionWriter
getAllDataProcessedFuture
in interface ResultPartitionWriter
public void onSubpartitionAllDataProcessed(int subpartition)
subpartition
- The index of the subpartition sending the notification.EndOfData
public void finish() throws IOException
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.
finish
in interface ResultPartitionWriter
IOException
public boolean isFinished()
isFinished
in interface ResultPartitionWriter
public void release()
public void release(Throwable cause)
ResultPartitionWriter
release
in interface ResultPartitionWriter
protected abstract void releaseInternal()
public void close()
ResultPartitionWriter
close
in interface AutoCloseable
close
in interface ResultPartitionWriter
public void fail(@Nullable Throwable throwable)
ResultPartitionWriter
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.
fail
in interface ResultPartitionWriter
throwable
- failure causepublic Throwable getFailureCause()
public int getNumTargetKeyGroups()
getNumTargetKeyGroups
in interface ResultPartitionWriter
public void setMetricGroup(TaskIOMetricGroup metrics)
ResultPartitionWriter
ResultPartitionWriter
.setMetricGroup
in interface ResultPartitionWriter
public boolean isReleased()
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.
isReleased
in interface ResultPartitionWriter
public CompletableFuture<?> getAvailableFuture()
getAvailableFuture
in interface AvailabilityProvider
protected void checkInProduceState() throws IllegalStateException
IllegalStateException
@VisibleForTesting public ResultPartitionManager getPartitionManager()
protected boolean canBeCompressed(Buffer buffer)
Copyright © 2014–2024 The Apache Software Foundation. All rights reserved.