Class AdaptiveScheduler
- java.lang.Object
-
- org.apache.flink.runtime.scheduler.adaptive.AdaptiveScheduler
-
- All Implemented Interfaces:
AutoCloseable
,StateTransitions
,StateTransitions.ToCancelling
,StateTransitions.ToCreatingExecutionGraph
,StateTransitions.ToExecuting
,StateTransitions.ToFailing
,StateTransitions.ToFinished
,StateTransitions.ToRestarting
,StateTransitions.ToStopWithSavepoint
,StateTransitions.ToWaitingForResources
,GlobalFailureHandler
,SchedulerNG
,org.apache.flink.util.AutoCloseableAsync
public class AdaptiveScheduler extends Object implements SchedulerNG
ASchedulerNG
implementation that uses the declarative resource management and automatically adapts the parallelism in case not enough resource could be acquired to run at the configured parallelism, as described in FLIP-160.This scheduler only supports jobs with streaming semantics, i.e., all vertices are connected via pipelined data-exchanges.
The implementation is spread over multiple
State
classes that control which RPCs are allowed in a given state and what state transitions are possible (see the FLIP for an overview). This class can thus be roughly split into 2 parts:1) RPCs, which must forward the call to the state via
State.tryRun(Class, ThrowingConsumer, String)
orState.tryCall(Class, FunctionWithException, String)
.2) Context methods, which are called by states, to either transition into another state or access functionality of some component in the scheduler.
-
-
Nested Class Summary
Nested Classes Modifier and Type Class Description static class
AdaptiveScheduler.Settings
Consolidated settings for the adaptive scheduler.-
Nested classes/interfaces inherited from interface org.apache.flink.runtime.scheduler.adaptive.StateTransitions
StateTransitions.ToCancelling, StateTransitions.ToCreatingExecutionGraph, StateTransitions.ToExecuting, StateTransitions.ToFailing, StateTransitions.ToFinished, StateTransitions.ToRestarting, StateTransitions.ToStopWithSavepoint, StateTransitions.ToWaitingForResources
-
-
Constructor Summary
Constructors Constructor Description AdaptiveScheduler(AdaptiveScheduler.Settings settings, JobGraph jobGraph, JobResourceRequirements jobResourceRequirements, org.apache.flink.configuration.Configuration configuration, DeclarativeSlotPool declarativeSlotPool, SlotAllocator slotAllocator, Executor ioExecutor, ClassLoader userCodeClassLoader, CheckpointsCleaner checkpointsCleaner, CheckpointRecoveryFactory checkpointRecoveryFactory, JobManagerJobMetricGroup jobManagerJobMetricGroup, RestartBackoffTimeStrategy restartBackoffTimeStrategy, long initializationTimestamp, org.apache.flink.runtime.concurrent.ComponentMainThreadExecutor mainThreadExecutor, org.apache.flink.runtime.rpc.FatalErrorHandler fatalErrorHandler, JobStatusListener jobStatusListener, Collection<org.apache.flink.core.failure.FailureEnricher> failureEnrichers, ExecutionGraphFactory executionGraphFactory)
-
Method Summary
All Methods Instance Methods Concrete Methods Modifier and Type Method Description void
acknowledgeCheckpoint(org.apache.flink.api.common.JobID jobID, ExecutionAttemptID executionAttemptID, long checkpointId, CheckpointMetrics checkpointMetrics, TaskStateSnapshot checkpointState)
void
archiveFailure(RootExceptionHistoryEntry failure)
void
cancel()
CompletableFuture<Void>
closeAsync()
void
declineCheckpoint(DeclineCheckpoint decline)
CompletableFuture<CoordinationResponse>
deliverCoordinationRequestToCoordinator(OperatorID operator, CoordinationRequest request)
Delivers a coordination request to theOperatorCoordinator
with the givenOperatorID
and returns the coordinator's response.void
deliverOperatorEventToCoordinator(ExecutionAttemptID taskExecution, OperatorID operator, OperatorEvent evt)
Delivers the given OperatorEvent to theOperatorCoordinator
with the givenOperatorID
.ArchivedExecutionGraph
getArchivedExecutionGraph(org.apache.flink.api.common.JobStatus jobStatus, Throwable cause)
Optional<VertexParallelism>
getAvailableVertexParallelism()
Executor
getIOExecutor()
org.apache.flink.api.common.JobID
getJobId()
CompletableFuture<org.apache.flink.api.common.JobStatus>
getJobTerminationFuture()
org.apache.flink.runtime.concurrent.ComponentMainThreadExecutor
getMainThreadExecutor()
JobManagerJobMetricGroup
getMetricGroup()
void
goToCanceling(ExecutionGraph executionGraph, ExecutionGraphHandler executionGraphHandler, OperatorCoordinatorHandler operatorCoordinatorHandler, List<ExceptionHistoryEntry> failureCollection)
Transitions into theCanceling
state.void
goToCreatingExecutionGraph(ExecutionGraph previousExecutionGraph)
Transitions into theCreatingExecutionGraph
state.void
goToExecuting(ExecutionGraph executionGraph, ExecutionGraphHandler executionGraphHandler, OperatorCoordinatorHandler operatorCoordinatorHandler, List<ExceptionHistoryEntry> failureCollection)
Transitions into theExecuting
state.void
goToFailing(ExecutionGraph executionGraph, ExecutionGraphHandler executionGraphHandler, OperatorCoordinatorHandler operatorCoordinatorHandler, Throwable failureCause, List<ExceptionHistoryEntry> failureCollection)
Transitions into theFailing
state.void
goToFinished(ArchivedExecutionGraph archivedExecutionGraph)
Transitions into theFinished
state.void
goToRestarting(ExecutionGraph executionGraph, ExecutionGraphHandler executionGraphHandler, OperatorCoordinatorHandler operatorCoordinatorHandler, Duration backoffTime, VertexParallelism restartWithParallelism, List<ExceptionHistoryEntry> failureCollection)
Transitions into theRestarting
state.CompletableFuture<String>
goToStopWithSavepoint(ExecutionGraph executionGraph, ExecutionGraphHandler executionGraphHandler, OperatorCoordinatorHandler operatorCoordinatorHandler, CheckpointScheduling checkpointScheduling, CompletableFuture<String> savepointFuture, List<ExceptionHistoryEntry> failureCollection)
Transitions into theStopWithSavepoint
state.void
goToWaitingForResources(ExecutionGraph previousExecutionGraph)
Transitions into theWaitingForResources
state.void
handleGlobalFailure(Throwable cause)
Handles a global failure.boolean
hasDesiredResources()
boolean
hasSufficientResources()
org.apache.flink.runtime.scheduler.adaptive.FailureResult
howToHandleFailure(Throwable failure, CompletableFuture<Map<String,String>> failureLabels)
boolean
isState(org.apache.flink.runtime.scheduler.adaptive.State expectedState)
void
notifyEndOfData(ExecutionAttemptID executionAttemptID)
Notifies that the task has reached the end of data.void
notifyKvStateRegistered(org.apache.flink.api.common.JobID jobId, JobVertexID jobVertexId, KeyGroupRange keyGroupRange, String registrationName, org.apache.flink.queryablestate.KvStateID kvStateId, InetSocketAddress kvStateServerAddress)
void
notifyKvStateUnregistered(org.apache.flink.api.common.JobID jobId, JobVertexID jobVertexId, KeyGroupRange keyGroupRange, String registrationName)
void
onFinished(ArchivedExecutionGraph archivedExecutionGraph)
void
reportCheckpointMetrics(org.apache.flink.api.common.JobID jobID, ExecutionAttemptID executionAttemptID, long checkpointId, CheckpointMetrics checkpointMetrics)
void
reportInitializationMetrics(org.apache.flink.api.common.JobID jobId, ExecutionAttemptID executionAttemptId, SubTaskInitializationMetrics initializationMetrics)
CheckpointStatsSnapshot
requestCheckpointStats()
Returns the checkpoint statistics for a given job.ExecutionGraphInfo
requestJob()
JobResourceRequirements
requestJobResourceRequirements()
Read currentjob resource requirements
.org.apache.flink.api.common.JobStatus
requestJobStatus()
KvStateLocation
requestKvStateLocation(org.apache.flink.api.common.JobID jobId, String registrationName)
SerializedInputSplit
requestNextInputSplit(JobVertexID vertexID, ExecutionAttemptID executionAttempt)
ExecutionState
requestPartitionState(IntermediateDataSetID intermediateResultId, ResultPartitionID resultPartitionId)
void
runIfState(org.apache.flink.runtime.scheduler.adaptive.State expectedState, Runnable action)
ScheduledFuture<?>
runIfState(org.apache.flink.runtime.scheduler.adaptive.State expectedState, Runnable action, Duration delay)
void
startScheduling()
CompletableFuture<String>
stopWithSavepoint(String targetDirectory, boolean terminate, org.apache.flink.core.execution.SavepointFormatType formatType)
CompletableFuture<CompletedCheckpoint>
triggerCheckpoint(org.apache.flink.core.execution.CheckpointType checkpointType)
CompletableFuture<String>
triggerSavepoint(String targetDirectory, boolean cancelJob, org.apache.flink.core.execution.SavepointFormatType formatType)
org.apache.flink.runtime.scheduler.adaptive.CreatingExecutionGraph.AssignmentResult
tryToAssignSlots(org.apache.flink.runtime.scheduler.adaptive.CreatingExecutionGraph.ExecutionGraphWithVertexParallelism executionGraphWithVertexParallelism)
void
updateAccumulators(AccumulatorSnapshot accumulatorSnapshot)
void
updateJobResourceRequirements(JobResourceRequirements jobResourceRequirements)
Updatejob resource requirements
.boolean
updateTaskExecutionState(TaskExecutionStateTransition taskExecutionState)
-
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.runtime.scheduler.SchedulerNG
updateTaskExecutionState
-
-
-
-
Constructor Detail
-
AdaptiveScheduler
public AdaptiveScheduler(AdaptiveScheduler.Settings settings, JobGraph jobGraph, @Nullable JobResourceRequirements jobResourceRequirements, org.apache.flink.configuration.Configuration configuration, DeclarativeSlotPool declarativeSlotPool, SlotAllocator slotAllocator, Executor ioExecutor, ClassLoader userCodeClassLoader, CheckpointsCleaner checkpointsCleaner, CheckpointRecoveryFactory checkpointRecoveryFactory, JobManagerJobMetricGroup jobManagerJobMetricGroup, RestartBackoffTimeStrategy restartBackoffTimeStrategy, long initializationTimestamp, org.apache.flink.runtime.concurrent.ComponentMainThreadExecutor mainThreadExecutor, org.apache.flink.runtime.rpc.FatalErrorHandler fatalErrorHandler, JobStatusListener jobStatusListener, Collection<org.apache.flink.core.failure.FailureEnricher> failureEnrichers, ExecutionGraphFactory executionGraphFactory) throws JobExecutionException
- Throws:
JobExecutionException
-
-
Method Detail
-
startScheduling
public void startScheduling()
- Specified by:
startScheduling
in interfaceSchedulerNG
-
closeAsync
public CompletableFuture<Void> closeAsync()
- Specified by:
closeAsync
in interfaceorg.apache.flink.util.AutoCloseableAsync
-
cancel
public void cancel()
- Specified by:
cancel
in interfaceSchedulerNG
-
getJobTerminationFuture
public CompletableFuture<org.apache.flink.api.common.JobStatus> getJobTerminationFuture()
- Specified by:
getJobTerminationFuture
in interfaceSchedulerNG
-
handleGlobalFailure
public void handleGlobalFailure(Throwable cause)
Description copied from interface:GlobalFailureHandler
Handles a global failure.- Specified by:
handleGlobalFailure
in interfaceGlobalFailureHandler
- Parameters:
cause
- A cause that describes the global failure.
-
updateTaskExecutionState
public boolean updateTaskExecutionState(TaskExecutionStateTransition taskExecutionState)
- Specified by:
updateTaskExecutionState
in interfaceSchedulerNG
-
requestNextInputSplit
public SerializedInputSplit requestNextInputSplit(JobVertexID vertexID, ExecutionAttemptID executionAttempt) throws IOException
- Specified by:
requestNextInputSplit
in interfaceSchedulerNG
- Throws:
IOException
-
requestPartitionState
public ExecutionState requestPartitionState(IntermediateDataSetID intermediateResultId, ResultPartitionID resultPartitionId) throws PartitionProducerDisposedException
- Specified by:
requestPartitionState
in interfaceSchedulerNG
- Throws:
PartitionProducerDisposedException
-
requestJob
public ExecutionGraphInfo requestJob()
- Specified by:
requestJob
in interfaceSchedulerNG
-
requestCheckpointStats
public CheckpointStatsSnapshot requestCheckpointStats()
Description copied from interface:SchedulerNG
Returns the checkpoint statistics for a given job. Although theCheckpointStatsSnapshot
is included in theExecutionGraphInfo
, this method is preferred toSchedulerNG.requestJob()
because it is less expensive.- Specified by:
requestCheckpointStats
in interfaceSchedulerNG
- Returns:
- checkpoint statistics snapshot for job graph
-
archiveFailure
public void archiveFailure(RootExceptionHistoryEntry failure)
-
requestJobStatus
public org.apache.flink.api.common.JobStatus requestJobStatus()
- Specified by:
requestJobStatus
in interfaceSchedulerNG
-
requestKvStateLocation
public KvStateLocation requestKvStateLocation(org.apache.flink.api.common.JobID jobId, String registrationName) throws UnknownKvStateLocation, FlinkJobNotFoundException
- Specified by:
requestKvStateLocation
in interfaceSchedulerNG
- Throws:
UnknownKvStateLocation
FlinkJobNotFoundException
-
notifyKvStateRegistered
public void notifyKvStateRegistered(org.apache.flink.api.common.JobID jobId, JobVertexID jobVertexId, KeyGroupRange keyGroupRange, String registrationName, org.apache.flink.queryablestate.KvStateID kvStateId, InetSocketAddress kvStateServerAddress) throws FlinkJobNotFoundException
- Specified by:
notifyKvStateRegistered
in interfaceSchedulerNG
- Throws:
FlinkJobNotFoundException
-
notifyKvStateUnregistered
public void notifyKvStateUnregistered(org.apache.flink.api.common.JobID jobId, JobVertexID jobVertexId, KeyGroupRange keyGroupRange, String registrationName) throws FlinkJobNotFoundException
- Specified by:
notifyKvStateUnregistered
in interfaceSchedulerNG
- Throws:
FlinkJobNotFoundException
-
updateAccumulators
public void updateAccumulators(AccumulatorSnapshot accumulatorSnapshot)
- Specified by:
updateAccumulators
in interfaceSchedulerNG
-
triggerSavepoint
public CompletableFuture<String> triggerSavepoint(@Nullable String targetDirectory, boolean cancelJob, org.apache.flink.core.execution.SavepointFormatType formatType)
- Specified by:
triggerSavepoint
in interfaceSchedulerNG
-
triggerCheckpoint
public CompletableFuture<CompletedCheckpoint> triggerCheckpoint(org.apache.flink.core.execution.CheckpointType checkpointType)
- Specified by:
triggerCheckpoint
in interfaceSchedulerNG
-
acknowledgeCheckpoint
public void acknowledgeCheckpoint(org.apache.flink.api.common.JobID jobID, ExecutionAttemptID executionAttemptID, long checkpointId, CheckpointMetrics checkpointMetrics, TaskStateSnapshot checkpointState)
- Specified by:
acknowledgeCheckpoint
in interfaceSchedulerNG
-
notifyEndOfData
public void notifyEndOfData(ExecutionAttemptID executionAttemptID)
Description copied from interface:SchedulerNG
Notifies that the task has reached the end of data.- Specified by:
notifyEndOfData
in interfaceSchedulerNG
- Parameters:
executionAttemptID
- The execution attempt id.
-
reportCheckpointMetrics
public void reportCheckpointMetrics(org.apache.flink.api.common.JobID jobID, ExecutionAttemptID executionAttemptID, long checkpointId, CheckpointMetrics checkpointMetrics)
- Specified by:
reportCheckpointMetrics
in interfaceSchedulerNG
-
declineCheckpoint
public void declineCheckpoint(DeclineCheckpoint decline)
- Specified by:
declineCheckpoint
in interfaceSchedulerNG
-
reportInitializationMetrics
public void reportInitializationMetrics(org.apache.flink.api.common.JobID jobId, ExecutionAttemptID executionAttemptId, SubTaskInitializationMetrics initializationMetrics)
- Specified by:
reportInitializationMetrics
in interfaceSchedulerNG
-
stopWithSavepoint
public CompletableFuture<String> stopWithSavepoint(@Nullable String targetDirectory, boolean terminate, org.apache.flink.core.execution.SavepointFormatType formatType)
- Specified by:
stopWithSavepoint
in interfaceSchedulerNG
-
deliverOperatorEventToCoordinator
public void deliverOperatorEventToCoordinator(ExecutionAttemptID taskExecution, OperatorID operator, OperatorEvent evt) throws org.apache.flink.util.FlinkException
Description copied from interface:SchedulerNG
Delivers the given OperatorEvent to theOperatorCoordinator
with the givenOperatorID
.Failure semantics: If the task manager sends an event for a non-running task or a non-existing operator coordinator, then respond with an exception to the call. If task and coordinator exist, then we assume that the call from the TaskManager was valid, and any bubbling exception needs to cause a job failure
- Specified by:
deliverOperatorEventToCoordinator
in interfaceSchedulerNG
- Throws:
org.apache.flink.util.FlinkException
- Thrown, if the task is not running or no operator/coordinator exists for the given ID.
-
deliverCoordinationRequestToCoordinator
public CompletableFuture<CoordinationResponse> deliverCoordinationRequestToCoordinator(OperatorID operator, CoordinationRequest request) throws org.apache.flink.util.FlinkException
Description copied from interface:SchedulerNG
Delivers a coordination request to theOperatorCoordinator
with the givenOperatorID
and returns the coordinator's response.- Specified by:
deliverCoordinationRequestToCoordinator
in interfaceSchedulerNG
- Returns:
- A future containing the response.
- Throws:
org.apache.flink.util.FlinkException
- Thrown, if the task is not running, or no operator/coordinator exists for the given ID, or the coordinator cannot handle client events.
-
requestJobResourceRequirements
public JobResourceRequirements requestJobResourceRequirements()
Description copied from interface:SchedulerNG
Read currentjob resource requirements
.- Specified by:
requestJobResourceRequirements
in interfaceSchedulerNG
- Returns:
- Current resource requirements.
-
updateJobResourceRequirements
public void updateJobResourceRequirements(JobResourceRequirements jobResourceRequirements)
Description copied from interface:SchedulerNG
Updatejob resource requirements
.- Specified by:
updateJobResourceRequirements
in interfaceSchedulerNG
- Parameters:
jobResourceRequirements
- new resource requirements
-
hasDesiredResources
public boolean hasDesiredResources()
-
hasSufficientResources
public boolean hasSufficientResources()
-
getJobId
public org.apache.flink.api.common.JobID getJobId()
-
getArchivedExecutionGraph
public ArchivedExecutionGraph getArchivedExecutionGraph(org.apache.flink.api.common.JobStatus jobStatus, @Nullable Throwable cause)
-
goToWaitingForResources
public void goToWaitingForResources(@Nullable ExecutionGraph previousExecutionGraph)
Description copied from interface:StateTransitions.ToWaitingForResources
Transitions into theWaitingForResources
state.- Specified by:
goToWaitingForResources
in interfaceStateTransitions.ToWaitingForResources
-
goToExecuting
public void goToExecuting(ExecutionGraph executionGraph, ExecutionGraphHandler executionGraphHandler, OperatorCoordinatorHandler operatorCoordinatorHandler, List<ExceptionHistoryEntry> failureCollection)
Description copied from interface:StateTransitions.ToExecuting
Transitions into theExecuting
state.- Specified by:
goToExecuting
in interfaceStateTransitions.ToExecuting
- Parameters:
executionGraph
- executionGraph to pass to theExecuting
stateexecutionGraphHandler
- executionGraphHandler to pass to theExecuting
stateoperatorCoordinatorHandler
- operatorCoordinatorHandler to pass to theExecuting
statefailureCollection
- collection of failures that are propagated
-
goToCanceling
public void goToCanceling(ExecutionGraph executionGraph, ExecutionGraphHandler executionGraphHandler, OperatorCoordinatorHandler operatorCoordinatorHandler, List<ExceptionHistoryEntry> failureCollection)
Description copied from interface:StateTransitions.ToCancelling
Transitions into theCanceling
state.- Specified by:
goToCanceling
in interfaceStateTransitions.ToCancelling
- Parameters:
executionGraph
- executionGraph to pass to theCanceling
stateexecutionGraphHandler
- executionGraphHandler to pass to theCanceling
stateoperatorCoordinatorHandler
- operatorCoordinatorHandler to pass to theCanceling
statefailureCollection
- collection of failures that are propagated
-
goToRestarting
public void goToRestarting(ExecutionGraph executionGraph, ExecutionGraphHandler executionGraphHandler, OperatorCoordinatorHandler operatorCoordinatorHandler, Duration backoffTime, @Nullable VertexParallelism restartWithParallelism, List<ExceptionHistoryEntry> failureCollection)
Description copied from interface:StateTransitions.ToRestarting
Transitions into theRestarting
state.- Specified by:
goToRestarting
in interfaceStateTransitions.ToRestarting
- Parameters:
executionGraph
- executionGraph to pass to theRestarting
stateexecutionGraphHandler
- executionGraphHandler to pass to theRestarting
stateoperatorCoordinatorHandler
- operatorCoordinatorHandler to pas to theRestarting
statebackoffTime
- backoffTime to wait before transitioning to theRestarting
staterestartWithParallelism
- theVertexParallelism
that triggered the restarting. TheAdaptiveScheduler
should transition directly toCreatingExecutionGraph
if the available parallelism hasn't changed while cancelling the job. Ifnull
is passed or the parallelism changed,WaitingForResources
state should be the subsequent state.failureCollection
- collection of failures that are propagated
-
goToFailing
public void goToFailing(ExecutionGraph executionGraph, ExecutionGraphHandler executionGraphHandler, OperatorCoordinatorHandler operatorCoordinatorHandler, Throwable failureCause, List<ExceptionHistoryEntry> failureCollection)
Description copied from interface:StateTransitions.ToFailing
Transitions into theFailing
state.- Specified by:
goToFailing
in interfaceStateTransitions.ToFailing
- Parameters:
executionGraph
- executionGraph to pass to theFailing
stateexecutionGraphHandler
- executionGraphHandler to pass to theFailing
stateoperatorCoordinatorHandler
- operatorCoordinatorHandler to pass to theFailing
statefailureCause
- failureCause describing why the job execution failedfailureCollection
- collection of failures that are propagated
-
goToStopWithSavepoint
public CompletableFuture<String> goToStopWithSavepoint(ExecutionGraph executionGraph, ExecutionGraphHandler executionGraphHandler, OperatorCoordinatorHandler operatorCoordinatorHandler, CheckpointScheduling checkpointScheduling, CompletableFuture<String> savepointFuture, List<ExceptionHistoryEntry> failureCollection)
Description copied from interface:StateTransitions.ToStopWithSavepoint
Transitions into theStopWithSavepoint
state.- Specified by:
goToStopWithSavepoint
in interfaceStateTransitions.ToStopWithSavepoint
- Parameters:
executionGraph
- executionGraph to pass to theStopWithSavepoint
stateexecutionGraphHandler
- executionGraphHandler to pass to theStopWithSavepoint
stateoperatorCoordinatorHandler
- operatorCoordinatorHandler to pass to theStopWithSavepoint
statesavepointFuture
- Future for the savepoint to complete.failureCollection
- collection of failures that are propagated- Returns:
- Location of the savepoint.
-
goToFinished
public void goToFinished(ArchivedExecutionGraph archivedExecutionGraph)
Description copied from interface:StateTransitions.ToFinished
Transitions into theFinished
state.- Specified by:
goToFinished
in interfaceStateTransitions.ToFinished
- Parameters:
archivedExecutionGraph
- archivedExecutionGraph which is passed to theFinished
state
-
goToCreatingExecutionGraph
public void goToCreatingExecutionGraph(@Nullable ExecutionGraph previousExecutionGraph)
Description copied from interface:StateTransitions.ToCreatingExecutionGraph
Transitions into theCreatingExecutionGraph
state.- Specified by:
goToCreatingExecutionGraph
in interfaceStateTransitions.ToCreatingExecutionGraph
-
tryToAssignSlots
public org.apache.flink.runtime.scheduler.adaptive.CreatingExecutionGraph.AssignmentResult tryToAssignSlots(org.apache.flink.runtime.scheduler.adaptive.CreatingExecutionGraph.ExecutionGraphWithVertexParallelism executionGraphWithVertexParallelism)
-
getAvailableVertexParallelism
public Optional<VertexParallelism> getAvailableVertexParallelism()
-
onFinished
public void onFinished(ArchivedExecutionGraph archivedExecutionGraph)
-
howToHandleFailure
public org.apache.flink.runtime.scheduler.adaptive.FailureResult howToHandleFailure(Throwable failure, CompletableFuture<Map<String,String>> failureLabels)
-
getIOExecutor
public Executor getIOExecutor()
-
getMainThreadExecutor
public org.apache.flink.runtime.concurrent.ComponentMainThreadExecutor getMainThreadExecutor()
-
getMetricGroup
public JobManagerJobMetricGroup getMetricGroup()
-
isState
public boolean isState(org.apache.flink.runtime.scheduler.adaptive.State expectedState)
-
runIfState
public void runIfState(org.apache.flink.runtime.scheduler.adaptive.State expectedState, Runnable action)
-
runIfState
public ScheduledFuture<?> runIfState(org.apache.flink.runtime.scheduler.adaptive.State expectedState, Runnable action, Duration delay)
-
-