Class JobMaster
- java.lang.Object
-
- org.apache.flink.runtime.rpc.RpcEndpoint
-
- org.apache.flink.runtime.rpc.FencedRpcEndpoint<JobMasterId>
-
- org.apache.flink.runtime.jobmaster.JobMaster
-
- All Implemented Interfaces:
AutoCloseable
,BlocklistListener
,CheckpointCoordinatorGateway
,JobMasterGateway
,JobMasterOperatorEventGateway
,JobMasterService
,KvStateLocationOracle
,KvStateRegistryGateway
,FencedRpcGateway<JobMasterId>
,RpcGateway
,AutoCloseableAsync
public class JobMaster extends FencedRpcEndpoint<JobMasterId> implements JobMasterGateway, JobMasterService
JobMaster implementation. The job master is responsible for the execution of a singleJobGraph
.It offers the following methods as part of its rpc interface to interact with the JobMaster remotely:
updateTaskExecutionState(org.apache.flink.runtime.taskmanager.TaskExecutionState)
updates the task execution state for given task
-
-
Nested Class Summary
-
Nested classes/interfaces inherited from class org.apache.flink.runtime.rpc.RpcEndpoint
RpcEndpoint.MainThreadExecutor
-
-
Field Summary
Fields Modifier and Type Field Description static String
JOB_MANAGER_NAME
Default names for Flink's distributed components.-
Fields inherited from class org.apache.flink.runtime.rpc.RpcEndpoint
log, rpcServer
-
-
Constructor Summary
Constructors Constructor Description JobMaster(RpcService rpcService, JobMasterId jobMasterId, JobMasterConfiguration jobMasterConfiguration, ResourceID resourceId, JobGraph jobGraph, HighAvailabilityServices highAvailabilityService, SlotPoolServiceSchedulerFactory slotPoolServiceSchedulerFactory, JobManagerSharedServices jobManagerSharedServices, HeartbeatServices heartbeatServices, JobManagerJobMetricGroupFactory jobMetricGroupFactory, OnCompletionActions jobCompletionActions, FatalErrorHandler fatalErrorHandler, ClassLoader userCodeLoader, ShuffleMaster<?> shuffleMaster, PartitionTrackerFactory partitionTrackerFactory, ExecutionDeploymentTracker executionDeploymentTracker, ExecutionDeploymentReconciler.Factory executionDeploymentReconcilerFactory, BlocklistHandler.Factory blocklistHandlerFactory, Collection<FailureEnricher> failureEnrichers, long initializationTimestamp)
-
Method Summary
All Methods Instance Methods Concrete Methods Modifier and Type Method Description void
acknowledgeCheckpoint(JobID jobID, ExecutionAttemptID executionAttemptID, long checkpointId, CheckpointMetrics checkpointMetrics, SerializedValue<TaskStateSnapshot> checkpointState)
CompletableFuture<Acknowledge>
cancel(Duration timeout)
Cancels the currently executed job.void
declineCheckpoint(DeclineCheckpoint decline)
CompletableFuture<CoordinationResponse>
deliverCoordinationRequestToCoordinator(OperatorID operatorId, SerializedValue<CoordinationRequest> serializedRequest, Duration timeout)
Deliver a coordination request to a specified coordinator and return the response.void
disconnectResourceManager(ResourceManagerId resourceManagerId, Exception cause)
Disconnects the resource manager from the job manager because of the given cause.CompletableFuture<Acknowledge>
disconnectTaskManager(ResourceID resourceID, Exception cause)
Disconnects the givenTaskExecutor
from theJobMaster
.void
failSlot(ResourceID taskManagerId, AllocationID allocationId, Exception cause)
Fails the slot with the given allocation id and cause.JobMasterGateway
getGateway()
Get theJobMasterGateway
belonging to this service.CompletableFuture<Collection<PartitionWithMetrics>>
getPartitionWithMetrics(Duration timeout, Set<ResultPartitionID> expectedPartitions)
Get specified partitions and their metrics (identified byexpectedPartitions
), the metrics include sizes of sub-partitions in a result partition.CompletableFuture<Void>
heartbeatFromResourceManager(ResourceID resourceID)
Sends heartbeat request from the resource manager.CompletableFuture<Void>
heartbeatFromTaskManager(ResourceID resourceID, TaskExecutorToJobManagerHeartbeatPayload payload)
Sends the heartbeat to job manager from task manager.void
notifyEndOfData(ExecutionAttemptID executionAttempt)
Notifies that the task has reached the end of data.CompletableFuture<Acknowledge>
notifyKvStateRegistered(JobID jobId, JobVertexID jobVertexId, KeyGroupRange keyGroupRange, String registrationName, KvStateID kvStateId, InetSocketAddress kvStateServerAddress)
Notifies that queryable state has been registered.CompletableFuture<Acknowledge>
notifyKvStateUnregistered(JobID jobId, JobVertexID jobVertexId, KeyGroupRange keyGroupRange, String registrationName)
Notifies that queryable state has been unregistered.CompletableFuture<Acknowledge>
notifyNewBlockedNodes(Collection<BlockedNode> newNodes)
Notify new blocked node records.void
notifyNotEnoughResourcesAvailable(Collection<ResourceRequirement> acquiredResources)
Notifies that not enough resources are available to fulfill the resource requirements of a job.CompletableFuture<Collection<SlotOffer>>
offerSlots(ResourceID taskManagerId, Collection<SlotOffer> slots, Duration timeout)
Offers the given slots to the job manager.protected void
onStart()
User overridable callback which is called fromRpcEndpoint.internalCallOnStart()
.CompletableFuture<Void>
onStop()
Suspend the job and shutdown all other services including rpc.CompletableFuture<RegistrationResponse>
registerTaskManager(JobID jobId, TaskManagerRegistrationInformation taskManagerRegistrationInformation, Duration timeout)
Registers the task manager at the job manager.void
reportCheckpointMetrics(JobID jobID, ExecutionAttemptID executionAttemptID, long checkpointId, CheckpointMetrics checkpointMetrics)
void
reportInitializationMetrics(JobID jobId, ExecutionAttemptID executionAttemptId, SubTaskInitializationMetrics initializationMetrics)
CompletableFuture<CheckpointStatsSnapshot>
requestCheckpointStats(Duration timeout)
Requests theCheckpointStatsSnapshot
of the job.CompletableFuture<ExecutionGraphInfo>
requestJob(Duration timeout)
Requests theExecutionGraphInfo
of the executed job.CompletableFuture<JobResourceRequirements>
requestJobResourceRequirements()
Read currentjob resource requirements
.CompletableFuture<JobStatus>
requestJobStatus(Duration timeout)
Requests the current job status.CompletableFuture<KvStateLocation>
requestKvStateLocation(JobID jobId, String registrationName)
Requests aKvStateLocation
for the specifiedInternalKvState
registration name.CompletableFuture<SerializedInputSplit>
requestNextInputSplit(JobVertexID vertexID, ExecutionAttemptID executionAttempt)
Requests the next input split for theExecutionJobVertex
.CompletableFuture<ExecutionState>
requestPartitionState(IntermediateDataSetID intermediateResultId, ResultPartitionID resultPartitionId)
Requests the current state of the partition.CompletableFuture<Acknowledge>
sendOperatorEventToCoordinator(ExecutionAttemptID task, OperatorID operatorID, SerializedValue<OperatorEvent> serializedEvent)
CompletableFuture<CoordinationResponse>
sendRequestToCoordinator(OperatorID operatorID, SerializedValue<CoordinationRequest> serializedRequest)
void
startFetchAndRetainPartitionWithMetricsOnTaskManager()
Notify jobMaster to fetch and retain partitions on task managers.CompletableFuture<?>
stopTrackingAndReleasePartitions(Collection<ResultPartitionID> partitionIds)
Notifies theJobMasterPartitionTracker
to stop tracking the target result partitions and release the locally occupied resources onTaskExecutor
s if any.CompletableFuture<String>
stopWithSavepoint(String targetDirectory, SavepointFormatType formatType, boolean terminate, Duration timeout)
Stops the job with a savepoint.CompletableFuture<CompletedCheckpoint>
triggerCheckpoint(CheckpointType checkpointType, Duration timeout)
Triggers taking a checkpoint of the executed job.CompletableFuture<String>
triggerSavepoint(String targetDirectory, boolean cancelJob, SavepointFormatType formatType, Duration timeout)
Triggers taking a savepoint of the executed job.CompletableFuture<Object>
updateGlobalAggregate(String aggregateName, Object aggregand, byte[] serializedAggregateFunction)
Update the aggregate and return the new value.CompletableFuture<Acknowledge>
updateJobResourceRequirements(JobResourceRequirements jobResourceRequirements)
Updatejob resource requirements
.CompletableFuture<Acknowledge>
updateTaskExecutionState(TaskExecutionState taskExecutionState)
Updates the task execution state for a given task.-
Methods inherited from class org.apache.flink.runtime.rpc.FencedRpcEndpoint
getFencingToken
-
Methods inherited from class org.apache.flink.runtime.rpc.RpcEndpoint
callAsync, closeAsync, getAddress, getEndpointId, getHostname, getMainThreadExecutor, getMainThreadExecutor, getRpcService, getSelfGateway, getTerminationFuture, internalCallOnStart, internalCallOnStop, isRunning, registerResource, runAsync, scheduleRunAsync, scheduleRunAsync, start, stop, unregisterResource, validateRunsInMainThread
-
Methods inherited from class java.lang.Object
clone, equals, finalize, getClass, hashCode, notify, notifyAll, toString, wait, wait, wait
-
Methods inherited from interface org.apache.flink.util.AutoCloseableAsync
close, closeAsync
-
Methods inherited from interface org.apache.flink.runtime.rpc.FencedRpcGateway
getFencingToken
-
Methods inherited from interface org.apache.flink.runtime.jobmaster.JobMasterGateway
triggerCheckpoint
-
Methods inherited from interface org.apache.flink.runtime.jobmaster.JobMasterService
getAddress, getTerminationFuture
-
Methods inherited from interface org.apache.flink.runtime.rpc.RpcGateway
getAddress, getHostname
-
-
-
-
Field Detail
-
JOB_MANAGER_NAME
public static final String JOB_MANAGER_NAME
Default names for Flink's distributed components.- See Also:
- Constant Field Values
-
-
Constructor Detail
-
JobMaster
public JobMaster(RpcService rpcService, JobMasterId jobMasterId, JobMasterConfiguration jobMasterConfiguration, ResourceID resourceId, JobGraph jobGraph, HighAvailabilityServices highAvailabilityService, SlotPoolServiceSchedulerFactory slotPoolServiceSchedulerFactory, JobManagerSharedServices jobManagerSharedServices, HeartbeatServices heartbeatServices, JobManagerJobMetricGroupFactory jobMetricGroupFactory, OnCompletionActions jobCompletionActions, FatalErrorHandler fatalErrorHandler, ClassLoader userCodeLoader, ShuffleMaster<?> shuffleMaster, PartitionTrackerFactory partitionTrackerFactory, ExecutionDeploymentTracker executionDeploymentTracker, ExecutionDeploymentReconciler.Factory executionDeploymentReconcilerFactory, BlocklistHandler.Factory blocklistHandlerFactory, Collection<FailureEnricher> failureEnrichers, long initializationTimestamp) throws Exception
- Throws:
Exception
-
-
Method Detail
-
onStart
protected void onStart() throws JobMasterException
Description copied from class:RpcEndpoint
User overridable callback which is called fromRpcEndpoint.internalCallOnStart()
.This method is called when the RpcEndpoint is being started. The method is guaranteed to be executed in the main thread context and can be used to start the rpc endpoint in the context of the rpc endpoint's main thread.
IMPORTANT: This method should never be called directly by the user.
- Overrides:
onStart
in classRpcEndpoint
- Throws:
JobMasterException
-
onStop
public CompletableFuture<Void> onStop()
Suspend the job and shutdown all other services including rpc.- Overrides:
onStop
in classRpcEndpoint
- Returns:
- Future which is completed once all post stop actions are completed. If an error occurs this future is completed exceptionally
-
cancel
public CompletableFuture<Acknowledge> cancel(Duration timeout)
Description copied from interface:JobMasterGateway
Cancels the currently executed job.- Specified by:
cancel
in interfaceJobMasterGateway
- Parameters:
timeout
- of this operation- Returns:
- Future acknowledge of the operation
-
updateTaskExecutionState
public CompletableFuture<Acknowledge> updateTaskExecutionState(TaskExecutionState taskExecutionState)
Updates the task execution state for a given task.- Specified by:
updateTaskExecutionState
in interfaceJobMasterGateway
- Parameters:
taskExecutionState
- New task execution state for a given task- Returns:
- Acknowledge the task execution state update
-
notifyEndOfData
public void notifyEndOfData(ExecutionAttemptID executionAttempt)
Description copied from interface:JobMasterGateway
Notifies that the task has reached the end of data.- Specified by:
notifyEndOfData
in interfaceJobMasterGateway
- Parameters:
executionAttempt
- The execution attempt id.
-
requestNextInputSplit
public CompletableFuture<SerializedInputSplit> requestNextInputSplit(JobVertexID vertexID, ExecutionAttemptID executionAttempt)
Description copied from interface:JobMasterGateway
Requests the next input split for theExecutionJobVertex
. The next input split is sent back to the sender as aSerializedInputSplit
message.- Specified by:
requestNextInputSplit
in interfaceJobMasterGateway
- Parameters:
vertexID
- The job vertex idexecutionAttempt
- The execution attempt id- Returns:
- The future of the input split. If there is no further input split, will return an empty object.
-
requestPartitionState
public CompletableFuture<ExecutionState> requestPartitionState(IntermediateDataSetID intermediateResultId, ResultPartitionID resultPartitionId)
Description copied from interface:JobMasterGateway
Requests the current state of the partition. The state of a partition is currently bound to the state of the producing execution.- Specified by:
requestPartitionState
in interfaceJobMasterGateway
- Parameters:
intermediateResultId
- The execution attempt ID of the task requesting the partition state.resultPartitionId
- The partition ID of the partition to request the state of.- Returns:
- The future of the partition state
-
disconnectTaskManager
public CompletableFuture<Acknowledge> disconnectTaskManager(ResourceID resourceID, Exception cause)
Description copied from interface:JobMasterGateway
Disconnects the givenTaskExecutor
from theJobMaster
.- Specified by:
disconnectTaskManager
in interfaceJobMasterGateway
- Parameters:
resourceID
- identifying the TaskManager to disconnectcause
- for the disconnection of the TaskManager- Returns:
- Future acknowledge once the JobMaster has been disconnected from the TaskManager
-
acknowledgeCheckpoint
public void acknowledgeCheckpoint(JobID jobID, ExecutionAttemptID executionAttemptID, long checkpointId, CheckpointMetrics checkpointMetrics, @Nullable SerializedValue<TaskStateSnapshot> checkpointState)
- Specified by:
acknowledgeCheckpoint
in interfaceCheckpointCoordinatorGateway
-
reportCheckpointMetrics
public void reportCheckpointMetrics(JobID jobID, ExecutionAttemptID executionAttemptID, long checkpointId, CheckpointMetrics checkpointMetrics)
- Specified by:
reportCheckpointMetrics
in interfaceCheckpointCoordinatorGateway
-
reportInitializationMetrics
public void reportInitializationMetrics(JobID jobId, ExecutionAttemptID executionAttemptId, SubTaskInitializationMetrics initializationMetrics)
- Specified by:
reportInitializationMetrics
in interfaceCheckpointCoordinatorGateway
-
declineCheckpoint
public void declineCheckpoint(DeclineCheckpoint decline)
- Specified by:
declineCheckpoint
in interfaceCheckpointCoordinatorGateway
-
sendOperatorEventToCoordinator
public CompletableFuture<Acknowledge> sendOperatorEventToCoordinator(ExecutionAttemptID task, OperatorID operatorID, SerializedValue<OperatorEvent> serializedEvent)
- Specified by:
sendOperatorEventToCoordinator
in interfaceJobMasterOperatorEventGateway
-
sendRequestToCoordinator
public CompletableFuture<CoordinationResponse> sendRequestToCoordinator(OperatorID operatorID, SerializedValue<CoordinationRequest> serializedRequest)
- Specified by:
sendRequestToCoordinator
in interfaceJobMasterOperatorEventGateway
-
requestKvStateLocation
public CompletableFuture<KvStateLocation> requestKvStateLocation(JobID jobId, String registrationName)
Description copied from interface:KvStateLocationOracle
Requests aKvStateLocation
for the specifiedInternalKvState
registration name.- Specified by:
requestKvStateLocation
in interfaceKvStateLocationOracle
- Parameters:
jobId
- identifying the job for which to request theKvStateLocation
registrationName
- Name under which the KvState has been registered.- Returns:
- Future of the requested
InternalKvState
location
-
notifyKvStateRegistered
public CompletableFuture<Acknowledge> notifyKvStateRegistered(JobID jobId, JobVertexID jobVertexId, KeyGroupRange keyGroupRange, String registrationName, KvStateID kvStateId, InetSocketAddress kvStateServerAddress)
Description copied from interface:KvStateRegistryGateway
Notifies that queryable state has been registered.- Specified by:
notifyKvStateRegistered
in interfaceKvStateRegistryGateway
- Parameters:
jobId
- identifying the job for which to register a key value statejobVertexId
- JobVertexID the KvState instance belongs to.keyGroupRange
- Key group range the KvState instance belongs to.registrationName
- Name under which the KvState has been registered.kvStateId
- ID of the registered KvState instance.kvStateServerAddress
- Server address where to find the KvState instance.- Returns:
- Future acknowledge if the key-value state has been registered
-
notifyKvStateUnregistered
public CompletableFuture<Acknowledge> notifyKvStateUnregistered(JobID jobId, JobVertexID jobVertexId, KeyGroupRange keyGroupRange, String registrationName)
Description copied from interface:KvStateRegistryGateway
Notifies that queryable state has been unregistered.- Specified by:
notifyKvStateUnregistered
in interfaceKvStateRegistryGateway
- Parameters:
jobId
- identifying the job for which to unregister a key value statejobVertexId
- JobVertexID the KvState instance belongs to.keyGroupRange
- Key group index the KvState instance belongs to.registrationName
- Name under which the KvState has been registered.- Returns:
- Future acknowledge if the key-value state has been unregistered
-
offerSlots
public CompletableFuture<Collection<SlotOffer>> offerSlots(ResourceID taskManagerId, Collection<SlotOffer> slots, Duration timeout)
Description copied from interface:JobMasterGateway
Offers the given slots to the job manager. The response contains the set of accepted slots.- Specified by:
offerSlots
in interfaceJobMasterGateway
- Parameters:
taskManagerId
- identifying the task managerslots
- to offer to the job managertimeout
- for the rpc call- Returns:
- Future set of accepted slots.
-
failSlot
public void failSlot(ResourceID taskManagerId, AllocationID allocationId, Exception cause)
Description copied from interface:JobMasterGateway
Fails the slot with the given allocation id and cause.- Specified by:
failSlot
in interfaceJobMasterGateway
- Parameters:
taskManagerId
- identifying the task managerallocationId
- identifying the slot to failcause
- of the failing
-
registerTaskManager
public CompletableFuture<RegistrationResponse> registerTaskManager(JobID jobId, TaskManagerRegistrationInformation taskManagerRegistrationInformation, Duration timeout)
Description copied from interface:JobMasterGateway
Registers the task manager at the job manager.- Specified by:
registerTaskManager
in interfaceJobMasterGateway
- Parameters:
jobId
- jobId specifying the job for which the JobMaster should be responsibletaskManagerRegistrationInformation
- the information for registering a task manager at the job managertimeout
- for the rpc call- Returns:
- Future registration response indicating whether the registration was successful or not
-
disconnectResourceManager
public void disconnectResourceManager(ResourceManagerId resourceManagerId, Exception cause)
Description copied from interface:JobMasterGateway
Disconnects the resource manager from the job manager because of the given cause.- Specified by:
disconnectResourceManager
in interfaceJobMasterGateway
- Parameters:
resourceManagerId
- identifying the resource manager leader idcause
- of the disconnect
-
heartbeatFromTaskManager
public CompletableFuture<Void> heartbeatFromTaskManager(ResourceID resourceID, TaskExecutorToJobManagerHeartbeatPayload payload)
Description copied from interface:JobMasterGateway
Sends the heartbeat to job manager from task manager.- Specified by:
heartbeatFromTaskManager
in interfaceJobMasterGateway
- Parameters:
resourceID
- unique id of the task managerpayload
- report payload- Returns:
- future which is completed exceptionally if the operation fails
-
heartbeatFromResourceManager
public CompletableFuture<Void> heartbeatFromResourceManager(ResourceID resourceID)
Description copied from interface:JobMasterGateway
Sends heartbeat request from the resource manager.- Specified by:
heartbeatFromResourceManager
in interfaceJobMasterGateway
- Parameters:
resourceID
- unique id of the resource manager- Returns:
- future which is completed exceptionally if the operation fails
-
requestJobStatus
public CompletableFuture<JobStatus> requestJobStatus(Duration timeout)
Description copied from interface:JobMasterGateway
Requests the current job status.- Specified by:
requestJobStatus
in interfaceJobMasterGateway
- Parameters:
timeout
- for the rpc call- Returns:
- Future containing the current job status
-
requestJob
public CompletableFuture<ExecutionGraphInfo> requestJob(Duration timeout)
Description copied from interface:JobMasterGateway
Requests theExecutionGraphInfo
of the executed job.- Specified by:
requestJob
in interfaceJobMasterGateway
- Parameters:
timeout
- for the rpc call- Returns:
- Future which is completed with the
ExecutionGraphInfo
of the executed job
-
requestCheckpointStats
public CompletableFuture<CheckpointStatsSnapshot> requestCheckpointStats(Duration timeout)
Description copied from interface:JobMasterGateway
Requests theCheckpointStatsSnapshot
of the job.- Specified by:
requestCheckpointStats
in interfaceJobMasterGateway
- Parameters:
timeout
- for the rpc call- Returns:
- Future which is completed with the
CheckpointStatsSnapshot
of the job
-
triggerCheckpoint
public CompletableFuture<CompletedCheckpoint> triggerCheckpoint(CheckpointType checkpointType, Duration timeout)
Description copied from interface:JobMasterGateway
Triggers taking a checkpoint of the executed job.- Specified by:
triggerCheckpoint
in interfaceJobMasterGateway
- Parameters:
checkpointType
- to determine how checkpoint should be takentimeout
- for the rpc call- Returns:
- Future which is completed with the CompletedCheckpoint once completed
-
triggerSavepoint
public CompletableFuture<String> triggerSavepoint(@Nullable String targetDirectory, boolean cancelJob, SavepointFormatType formatType, Duration timeout)
Description copied from interface:JobMasterGateway
Triggers taking a savepoint of the executed job.- Specified by:
triggerSavepoint
in interfaceJobMasterGateway
- Parameters:
targetDirectory
- to which to write the savepoint data or null if the default savepoint directory should be usedformatType
- binary format for the savepointtimeout
- for the rpc call- Returns:
- Future which is completed with the savepoint path once completed
-
stopWithSavepoint
public CompletableFuture<String> stopWithSavepoint(@Nullable String targetDirectory, SavepointFormatType formatType, boolean terminate, Duration timeout)
Description copied from interface:JobMasterGateway
Stops the job with a savepoint.- Specified by:
stopWithSavepoint
in interfaceJobMasterGateway
- Parameters:
targetDirectory
- to which to write the savepoint data or null if the default savepoint directory should be usedterminate
- flag indicating if the job should terminate or just suspendtimeout
- for the rpc call- Returns:
- Future which is completed with the savepoint path once completed
-
notifyNotEnoughResourcesAvailable
public void notifyNotEnoughResourcesAvailable(Collection<ResourceRequirement> acquiredResources)
Description copied from interface:JobMasterGateway
Notifies that not enough resources are available to fulfill the resource requirements of a job.- Specified by:
notifyNotEnoughResourcesAvailable
in interfaceJobMasterGateway
- Parameters:
acquiredResources
- the resources that have been acquired for the job
-
updateGlobalAggregate
public CompletableFuture<Object> updateGlobalAggregate(String aggregateName, Object aggregand, byte[] serializedAggregateFunction)
Description copied from interface:JobMasterGateway
Update the aggregate and return the new value.- Specified by:
updateGlobalAggregate
in interfaceJobMasterGateway
- Parameters:
aggregateName
- The name of the aggregate to updateaggregand
- The value to add to the aggregateserializedAggregateFunction
- The function to apply to the current aggregate and aggregand to obtain the new aggregate value, this should be of typeAggregateFunction
- Returns:
- The updated aggregate
-
deliverCoordinationRequestToCoordinator
public CompletableFuture<CoordinationResponse> deliverCoordinationRequestToCoordinator(OperatorID operatorId, SerializedValue<CoordinationRequest> serializedRequest, Duration timeout)
Description copied from interface:JobMasterGateway
Deliver a coordination request to a specified coordinator and return the response.- Specified by:
deliverCoordinationRequestToCoordinator
in interfaceJobMasterGateway
- Parameters:
operatorId
- identifying the coordinator to receive the requestserializedRequest
- serialized request to deliver- Returns:
- A future containing the response. The response will fail with a
FlinkException
if the task is not running, or no operator/coordinator exists for the given ID, or the coordinator cannot handle client events.
-
stopTrackingAndReleasePartitions
public CompletableFuture<?> stopTrackingAndReleasePartitions(Collection<ResultPartitionID> partitionIds)
Description copied from interface:JobMasterGateway
Notifies theJobMasterPartitionTracker
to stop tracking the target result partitions and release the locally occupied resources onTaskExecutor
s if any.- Specified by:
stopTrackingAndReleasePartitions
in interfaceJobMasterGateway
-
getPartitionWithMetrics
public CompletableFuture<Collection<PartitionWithMetrics>> getPartitionWithMetrics(Duration timeout, Set<ResultPartitionID> expectedPartitions)
Description copied from interface:JobMasterGateway
Get specified partitions and their metrics (identified byexpectedPartitions
), the metrics include sizes of sub-partitions in a result partition.- Specified by:
getPartitionWithMetrics
in interfaceJobMasterGateway
- Parameters:
timeout
- 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.
-
startFetchAndRetainPartitionWithMetricsOnTaskManager
public void startFetchAndRetainPartitionWithMetricsOnTaskManager()
Description copied from interface:JobMasterGateway
Notify jobMaster to fetch and retain partitions on task managers. It will process for future TaskManager registrations and already registered TaskManagers.- Specified by:
startFetchAndRetainPartitionWithMetricsOnTaskManager
in interfaceJobMasterGateway
-
notifyNewBlockedNodes
public CompletableFuture<Acknowledge> notifyNewBlockedNodes(Collection<BlockedNode> newNodes)
Description copied from interface:BlocklistListener
Notify new blocked node records.- Specified by:
notifyNewBlockedNodes
in interfaceBlocklistListener
- Parameters:
newNodes
- the new blocked node records- Returns:
- Future acknowledge once the new nodes have successfully notified.
-
requestJobResourceRequirements
public CompletableFuture<JobResourceRequirements> requestJobResourceRequirements()
Description copied from interface:JobMasterGateway
Read currentjob resource requirements
.- Specified by:
requestJobResourceRequirements
in interfaceJobMasterGateway
- Returns:
- Future which that contains current resource requirements.
-
updateJobResourceRequirements
public CompletableFuture<Acknowledge> updateJobResourceRequirements(JobResourceRequirements jobResourceRequirements)
Description copied from interface:JobMasterGateway
Updatejob resource requirements
.- Specified by:
updateJobResourceRequirements
in interfaceJobMasterGateway
- Parameters:
jobResourceRequirements
- new resource requirements- Returns:
- Future which is completed successfully when requirements are updated
-
getGateway
public JobMasterGateway getGateway()
Description copied from interface:JobMasterService
Get theJobMasterGateway
belonging to this service.- Specified by:
getGateway
in interfaceJobMasterService
- Returns:
- JobMasterGateway belonging to this service
-
-