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 void
close()
Closes this shuffle master service which should release all resources.default MemorySize
computeShuffleMemorySizeForTask(TaskInputsOutputsDescriptor taskInputsOutputsDescriptor)
Compute shuffle memory size for a task with the givenTaskInputsOutputsDescriptor
.default CompletableFuture<Collection<PartitionWithMetrics>>
getPartitionWithMetrics(JobID jobId, Duration timeout, Set<ResultPartitionID> expectedPartitions)
Retrieves specified partitions and their metrics (identified byexpectedPartitions
), the metrics include sizes of sub-partitions in a result partition.default void
notifyPartitionRecoveryStarted(JobID jobId)
Notifies that the recovery process of result partitions has started.default void
registerJob(JobShuffleContext context)
Registers the target job together with the correspondingJobShuffleContext
to this shuffle master.CompletableFuture<T>
registerPartitionWithProducer(JobID jobID, PartitionDescriptor partitionDescriptor, ProducerDescriptor producerDescriptor)
Asynchronously register a partition and its producer with the shuffle service.void
releasePartitionExternally(ShuffleDescriptor shuffleDescriptor)
Release any external resources occupied by the given partition.default void
restoreState(List<ShuffleMasterSnapshot> snapshots)
Restores the state of the shuffle master from the provided snapshots.default void
snapshotState(CompletableFuture<ShuffleMasterSnapshot> snapshotFuture, ShuffleMasterSnapshotContext context)
Triggers a snapshot of the shuffle master's state.default void
start()
Starts this shuffle master as a service.default boolean
supportsBatchSnapshot()
Whether the shuffle master supports taking snapshot in batch scenarios ifBatchExecutionOptions.JOB_RECOVERY_ENABLED
is true.default void
unregisterJob(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 Exception
Starts 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 Exception
Closes this shuffle master service which should release all resources. A shuffle master will only be closed when the cluster is shut down.- Specified by:
close
in interfaceAutoCloseable
- Throws:
Exception
-
registerJob
default void registerJob(JobShuffleContext context)
Registers the target job together with the correspondingJobShuffleContext
to 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(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(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 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
.
-
getPartitionWithMetrics
default CompletableFuture<Collection<PartitionWithMetrics>> getPartitionWithMetrics(JobID jobId, Duration timeout, Set<ResultPartitionID> expectedPartitions)
Retrieves specified partitions and their metrics (identified byexpectedPartitions
), the metrics include sizes of sub-partitions in a result partition.- Parameters:
jobId
- ID of the target jobtimeout
- The timeout used for retrieve the specified partitions.expectedPartitions
- The set of identifiers for the result partitions whose metrics are to be fetched.- Returns:
- A future will contain a collection of the partitions with their metrics that could be retrieved from the expected partitions within the specified timeout period.
-
supportsBatchSnapshot
default boolean supportsBatchSnapshot()
Whether the shuffle master supports taking snapshot in batch scenarios ifBatchExecutionOptions.JOB_RECOVERY_ENABLED
is true. If it returns true, Flink will callsnapshotState(java.util.concurrent.CompletableFuture<org.apache.flink.runtime.shuffle.ShuffleMasterSnapshot>, org.apache.flink.runtime.shuffle.ShuffleMasterSnapshotContext)
to take snapshot, and callrestoreState(java.util.List<org.apache.flink.runtime.shuffle.ShuffleMasterSnapshot>)
to restore the state of shuffle master.
-
snapshotState
default void snapshotState(CompletableFuture<ShuffleMasterSnapshot> snapshotFuture, ShuffleMasterSnapshotContext context)
Triggers a snapshot of the shuffle master's state.
-
restoreState
default void restoreState(List<ShuffleMasterSnapshot> snapshots)
Restores the state of the shuffle master from the provided snapshots.
-
notifyPartitionRecoveryStarted
default void notifyPartitionRecoveryStarted(JobID jobId)
Notifies that the recovery process of result partitions has started.- Parameters:
jobId
- ID of the target job
-
-