Package org.apache.flink.runtime.shuffle
Interface ShuffleMaster<T extends ShuffleDescriptor>
-
- Type Parameters:
T- partition shuffle descriptor used for producer/consumer deployment and their data exchange.
- All Superinterfaces:
AutoCloseable
- All Known Implementing Classes:
NettyShuffleMaster
public interface ShuffleMaster<T extends ShuffleDescriptor> extends AutoCloseable
Intermediate result partition registry to use inJobMaster.
-
-
Method Summary
All Methods Instance Methods Abstract Methods Default Methods Modifier and Type Method Description default voidclose()Closes this shuffle master service which should release all resources.default org.apache.flink.configuration.MemorySizecomputeShuffleMemorySizeForTask(TaskInputsOutputsDescriptor taskInputsOutputsDescriptor)Compute shuffle memory size for a task with the givenTaskInputsOutputsDescriptor.default voidregisterJob(JobShuffleContext context)Registers the target job together with the correspondingJobShuffleContextto this shuffle master.CompletableFuture<T>registerPartitionWithProducer(org.apache.flink.api.common.JobID jobID, PartitionDescriptor partitionDescriptor, ProducerDescriptor producerDescriptor)Asynchronously register a partition and its producer with the shuffle service.voidreleasePartitionExternally(ShuffleDescriptor shuffleDescriptor)Release any external resources occupied by the given partition.default voidstart()Starts this shuffle master as a service.default voidunregisterJob(org.apache.flink.api.common.JobID jobID)Unregisters the target job from this shuffle master, which means the corresponding job has reached a global termination state and all the allocated resources except for the cluster partitions can be cleared.
-
-
-
Method Detail
-
start
default void start() throws ExceptionStarts this shuffle master as a service. One can do some initialization here, for example getting access and connecting to the external system.- Throws:
Exception
-
close
default void close() throws ExceptionCloses this shuffle master service which should release all resources. A shuffle master will only be closed when the cluster is shut down.- Specified by:
closein interfaceAutoCloseable- Throws:
Exception
-
registerJob
default void registerJob(JobShuffleContext context)
Registers the target job together with the correspondingJobShuffleContextto this shuffle master. Through the shuffle context, one can obtain some basic information like job ID, job configuration. It enables ShuffleMaster to notify JobMaster about lost result partitions, so that JobMaster can identify and reproduce unavailable partitions earlier.- Parameters:
context- the corresponding shuffle context of the target job.
-
unregisterJob
default void unregisterJob(org.apache.flink.api.common.JobID jobID)
Unregisters the target job from this shuffle master, which means the corresponding job has reached a global termination state and all the allocated resources except for the cluster partitions can be cleared.- Parameters:
jobID- ID of the target job to be unregistered.
-
registerPartitionWithProducer
CompletableFuture<T> registerPartitionWithProducer(org.apache.flink.api.common.JobID jobID, PartitionDescriptor partitionDescriptor, ProducerDescriptor producerDescriptor)
Asynchronously register a partition and its producer with the shuffle service.The returned shuffle descriptor is an internal handle which identifies the partition internally within the shuffle service. The descriptor should provide enough information to read from or write data to the partition.
- Parameters:
jobID- job ID of the corresponding job which registered the partitionpartitionDescriptor- general job graph information about the partitionproducerDescriptor- general producer information (location, execution id, connection info)- Returns:
- future with the partition shuffle descriptor used for producer/consumer deployment and their data exchange.
-
releasePartitionExternally
void releasePartitionExternally(ShuffleDescriptor shuffleDescriptor)
Release any external resources occupied by the given partition.This call triggers release of any resources which are occupied by the given partition in the external systems outside of the producer executor. This is mostly relevant for the batch jobs and blocking result partitions. The producer local resources are managed by
ShuffleDescriptor.storesLocalResourcesOn()andShuffleEnvironment.releasePartitionsLocally(Collection).- Parameters:
shuffleDescriptor- shuffle descriptor of the result partition to release externally.
-
computeShuffleMemorySizeForTask
default org.apache.flink.configuration.MemorySize computeShuffleMemorySizeForTask(TaskInputsOutputsDescriptor taskInputsOutputsDescriptor)
Compute shuffle memory size for a task with the givenTaskInputsOutputsDescriptor.- Parameters:
taskInputsOutputsDescriptor- describes task inputs and outputs information for shuffle memory calculation.- Returns:
- shuffle memory size for a task with the given
TaskInputsOutputsDescriptor.
-
-