Interface TaskExecutorGateway
-
- All Superinterfaces:
org.apache.flink.runtime.rpc.RpcGateway
,TaskExecutorOperatorEventGateway
,TaskExecutorThreadInfoGateway
- All Known Implementing Classes:
TaskExecutor
,TaskExecutorGatewayDecoratorBase
public interface TaskExecutorGateway extends org.apache.flink.runtime.rpc.RpcGateway, TaskExecutorOperatorEventGateway, TaskExecutorThreadInfoGateway
TaskExecutor
RPC gateway interface.
-
-
Method Summary
All Methods Instance Methods Abstract Methods Default Methods Modifier and Type Method Description CompletableFuture<Acknowledge>
abortCheckpoint(ExecutionAttemptID executionAttemptID, long checkpointId, long latestCompletedCheckpointId, long checkpointTimestamp)
Abort a checkpoint for the given task.CompletableFuture<Boolean>
canBeReleased()
Checks whether the task executor can be released.CompletableFuture<Acknowledge>
cancelTask(ExecutionAttemptID executionAttemptID, Duration timeout)
Cancel the given task.CompletableFuture<Acknowledge>
confirmCheckpoint(ExecutionAttemptID executionAttemptID, long completedCheckpointId, long completedCheckpointTimestamp, long lastSubsumedCheckpointId)
Confirm a checkpoint for the given task.void
disconnectJobManager(org.apache.flink.api.common.JobID jobId, Exception cause)
Disconnects the given JobManager from the TaskManager.void
disconnectResourceManager(Exception cause)
Disconnects the ResourceManager from the TaskManager.void
freeInactiveSlots(org.apache.flink.api.common.JobID jobId, Duration timeout)
Frees all currently inactive slot allocated for the given job.CompletableFuture<Acknowledge>
freeSlot(AllocationID allocationId, Throwable cause, Duration timeout)
Frees the slot with the given allocation ID.default CompletableFuture<Collection<PartitionWithMetrics>>
getAndRetainPartitionWithMetrics(org.apache.flink.api.common.JobID jobId)
Get and retain all partitions and their metrics located on this task executor, the metrics mainly includes the meta information of partition(partition bytes, etc).CompletableFuture<Void>
heartbeatFromJobManager(ResourceID heartbeatOrigin, AllocatedSlotReport allocatedSlotReport)
Heartbeat request from the job manager.CompletableFuture<Void>
heartbeatFromResourceManager(ResourceID heartbeatOrigin)
Heartbeat request from the resource manager.CompletableFuture<Acknowledge>
promotePartitions(org.apache.flink.api.common.JobID jobId, Set<ResultPartitionID> partitionIds)
Batch promote intermediate result partitions.CompletableFuture<Acknowledge>
releaseClusterPartitions(Collection<IntermediateDataSetID> dataSetsToRelease, Duration timeout)
Releases all cluster partitions belong to any of the given data sets.void
releasePartitions(org.apache.flink.api.common.JobID jobId, Set<ResultPartitionID> partitionIds)
Batch release intermediate result partitions.CompletableFuture<TransientBlobKey>
requestFileUploadByName(String fileName, Duration timeout)
Requests the file upload of the specified name to the cluster'sBlobServer
.CompletableFuture<TransientBlobKey>
requestFileUploadByNameAndType(String fileName, FileType fileType, Duration timeout)
Requests the file upload of the specified name and file type to the cluster'sBlobServer
.CompletableFuture<TransientBlobKey>
requestFileUploadByType(FileType fileType, Duration timeout)
Requests the file upload of the specified type to the cluster'sBlobServer
.CompletableFuture<Collection<LogInfo>>
requestLogList(Duration timeout)
Requests for the historical log file names on the TaskManager.CompletableFuture<org.apache.flink.types.SerializableOptional<String>>
requestMetricQueryServiceAddress(Duration timeout)
Returns the gateway of Metric Query Service on the TaskManager.CompletableFuture<ProfilingInfo>
requestProfiling(int duration, ProfilingInfo.ProfilingMode mode, Duration timeout)
Requests the profiling from this TaskManager.CompletableFuture<Collection<ProfilingInfo>>
requestProfilingList(Duration timeout)
Requests for the historical profiling file names on the TaskManager.CompletableFuture<Acknowledge>
requestSlot(SlotID slotId, org.apache.flink.api.common.JobID jobId, AllocationID allocationId, ResourceProfile resourceProfile, String targetAddress, ResourceManagerId resourceManagerId, Duration timeout)
Requests a slot from the TaskManager.CompletableFuture<ThreadDumpInfo>
requestThreadDump(Duration timeout)
Requests the thread dump from this TaskManager.CompletableFuture<Acknowledge>
sendOperatorEventToTask(ExecutionAttemptID task, OperatorID operator, org.apache.flink.util.SerializedValue<OperatorEvent> evt)
Sends an operator event to an operator in a task executed by the Task Manager (Task Executor).CompletableFuture<Acknowledge>
submitTask(TaskDeploymentDescriptor tdd, JobMasterId jobMasterId, Duration timeout)
Submit aTask
to theTaskExecutor
.CompletableFuture<Acknowledge>
triggerCheckpoint(ExecutionAttemptID executionAttemptID, long checkpointID, long checkpointTimestamp, CheckpointOptions checkpointOptions)
Trigger the checkpoint for the given task.CompletableFuture<Acknowledge>
updateDelegationTokens(ResourceManagerId resourceManagerId, byte[] tokens)
Sends new delegation tokens to this TaskManager.CompletableFuture<Acknowledge>
updatePartitions(ExecutionAttemptID executionAttemptID, Iterable<PartitionInfo> partitionInfos, Duration timeout)
Update the task where the given partitions can be found.-
Methods inherited from interface org.apache.flink.runtime.taskexecutor.TaskExecutorThreadInfoGateway
requestThreadInfoSamples
-
-
-
-
Method Detail
-
requestSlot
CompletableFuture<Acknowledge> requestSlot(SlotID slotId, org.apache.flink.api.common.JobID jobId, AllocationID allocationId, ResourceProfile resourceProfile, String targetAddress, ResourceManagerId resourceManagerId, Duration timeout)
Requests a slot from the TaskManager.- Parameters:
slotId
- slot id for the requestjobId
- for which to request a slotallocationId
- id for the requestresourceProfile
- of requested slot, used only for dynamic slot allocation and will be ignored otherwisetargetAddress
- to which to offer the requested slotsresourceManagerId
- current leader id of the ResourceManagertimeout
- for the operation- Returns:
- answer to the slot request
-
submitTask
CompletableFuture<Acknowledge> submitTask(TaskDeploymentDescriptor tdd, JobMasterId jobMasterId, Duration timeout)
Submit aTask
to theTaskExecutor
.- Parameters:
tdd
- describing the task to submitjobMasterId
- identifying the submitting JobMastertimeout
- of the submit operation- Returns:
- Future acknowledge of the successful operation
-
updatePartitions
CompletableFuture<Acknowledge> updatePartitions(ExecutionAttemptID executionAttemptID, Iterable<PartitionInfo> partitionInfos, Duration timeout)
Update the task where the given partitions can be found.- Parameters:
executionAttemptID
- identifying the taskpartitionInfos
- telling where the partition can be retrieved fromtimeout
- for the update partitions operation- Returns:
- Future acknowledge if the partitions have been successfully updated
-
releasePartitions
void releasePartitions(org.apache.flink.api.common.JobID jobId, Set<ResultPartitionID> partitionIds)
Batch release intermediate result partitions.- Parameters:
jobId
- id of the job that the partitions belong topartitionIds
- partition ids to release
-
promotePartitions
CompletableFuture<Acknowledge> promotePartitions(org.apache.flink.api.common.JobID jobId, Set<ResultPartitionID> partitionIds)
Batch promote intermediate result partitions.- Parameters:
jobId
- id of the job that the partitions belong topartitionIds
- partition ids to release- Returns:
- Future acknowledge that the partitions are successfully promoted.
-
releaseClusterPartitions
CompletableFuture<Acknowledge> releaseClusterPartitions(Collection<IntermediateDataSetID> dataSetsToRelease, Duration timeout)
Releases all cluster partitions belong to any of the given data sets.- Parameters:
dataSetsToRelease
- data sets for which all cluster partitions should be releasedtimeout
- for the partitions release operation- Returns:
- Future acknowledge that the request was received
-
triggerCheckpoint
CompletableFuture<Acknowledge> triggerCheckpoint(ExecutionAttemptID executionAttemptID, long checkpointID, long checkpointTimestamp, CheckpointOptions checkpointOptions)
Trigger the checkpoint for the given task. The checkpoint is identified by the checkpoint ID and the checkpoint timestamp.- Parameters:
executionAttemptID
- identifying the taskcheckpointID
- unique id for the checkpointcheckpointTimestamp
- is the timestamp when the checkpoint has been initiatedcheckpointOptions
- for performing the checkpoint- Returns:
- Future acknowledge if the checkpoint has been successfully triggered
-
confirmCheckpoint
CompletableFuture<Acknowledge> confirmCheckpoint(ExecutionAttemptID executionAttemptID, long completedCheckpointId, long completedCheckpointTimestamp, long lastSubsumedCheckpointId)
Confirm a checkpoint for the given task. The checkpoint is identified by the checkpoint ID and the checkpoint timestamp.- Parameters:
executionAttemptID
- identifying the taskcompletedCheckpointId
- unique id for the completed checkpointcompletedCheckpointTimestamp
- is the timestamp when the checkpoint has been initiatedlastSubsumedCheckpointId
- unique id for the checkpoint to be subsumed- Returns:
- Future acknowledge if the checkpoint has been successfully confirmed
-
abortCheckpoint
CompletableFuture<Acknowledge> abortCheckpoint(ExecutionAttemptID executionAttemptID, long checkpointId, long latestCompletedCheckpointId, long checkpointTimestamp)
Abort a checkpoint for the given task. The checkpoint is identified by the checkpoint ID and the checkpoint timestamp.- Parameters:
executionAttemptID
- identifying the taskcheckpointId
- unique id for the checkpointlatestCompletedCheckpointId
- the id of the latest completed checkpointcheckpointTimestamp
- is the timestamp when the checkpoint has been initiated- Returns:
- Future acknowledge if the checkpoint has been successfully confirmed
-
cancelTask
CompletableFuture<Acknowledge> cancelTask(ExecutionAttemptID executionAttemptID, Duration timeout)
Cancel the given task.- Parameters:
executionAttemptID
- identifying the tasktimeout
- for the cancel operation- Returns:
- Future acknowledge if the task is successfully canceled
-
heartbeatFromJobManager
CompletableFuture<Void> heartbeatFromJobManager(ResourceID heartbeatOrigin, AllocatedSlotReport allocatedSlotReport)
Heartbeat request from the job manager.- Parameters:
heartbeatOrigin
- unique id of the job manager- Returns:
- future which is completed exceptionally if the operation fails
-
heartbeatFromResourceManager
CompletableFuture<Void> heartbeatFromResourceManager(ResourceID heartbeatOrigin)
Heartbeat request from the resource manager.- Parameters:
heartbeatOrigin
- unique id of the resource manager- Returns:
- future which is completed exceptionally if the operation fails
-
disconnectJobManager
void disconnectJobManager(org.apache.flink.api.common.JobID jobId, Exception cause)
Disconnects the given JobManager from the TaskManager.- Parameters:
jobId
- JobID for which the JobManager was the leadercause
- for the disconnection from the JobManager
-
disconnectResourceManager
void disconnectResourceManager(Exception cause)
Disconnects the ResourceManager from the TaskManager.- Parameters:
cause
- for the disconnection from the ResourceManager
-
freeSlot
CompletableFuture<Acknowledge> freeSlot(AllocationID allocationId, Throwable cause, Duration timeout)
Frees the slot with the given allocation ID.- Parameters:
allocationId
- identifying the slot to freecause
- of the freeing operationtimeout
- for the operation- Returns:
- Future acknowledge which is returned once the slot has been freed
-
freeInactiveSlots
void freeInactiveSlots(org.apache.flink.api.common.JobID jobId, Duration timeout)
Frees all currently inactive slot allocated for the given job.- Parameters:
jobId
- job for which all inactive slots should be releasedtimeout
- for the operation
-
requestFileUploadByType
CompletableFuture<TransientBlobKey> requestFileUploadByType(FileType fileType, Duration timeout)
Requests the file upload of the specified type to the cluster'sBlobServer
.- Parameters:
fileType
- to uploadtimeout
- for the asynchronous operation- Returns:
- Future which is completed with the
TransientBlobKey
of the uploaded file.
-
requestFileUploadByName
CompletableFuture<TransientBlobKey> requestFileUploadByName(String fileName, Duration timeout)
Requests the file upload of the specified name to the cluster'sBlobServer
.- Parameters:
fileName
- to uploadtimeout
- for the asynchronous operation- Returns:
- Future which is completed with the
TransientBlobKey
of the uploaded file.
-
requestFileUploadByNameAndType
CompletableFuture<TransientBlobKey> requestFileUploadByNameAndType(String fileName, FileType fileType, Duration timeout)
Requests the file upload of the specified name and file type to the cluster'sBlobServer
.- Parameters:
fileName
- to uploadfileType
- to uploadtimeout
- for the asynchronous operation- Returns:
- Future which is completed with the
TransientBlobKey
of the uploaded file.
-
requestMetricQueryServiceAddress
CompletableFuture<org.apache.flink.types.SerializableOptional<String>> requestMetricQueryServiceAddress(Duration timeout)
Returns the gateway of Metric Query Service on the TaskManager.- Returns:
- Future gateway of Metric Query Service on the TaskManager.
-
canBeReleased
CompletableFuture<Boolean> canBeReleased()
Checks whether the task executor can be released. It cannot be released if there're unconsumed result partitions.- Returns:
- Future flag indicating whether the task executor can be released.
-
requestLogList
CompletableFuture<Collection<LogInfo>> requestLogList(Duration timeout)
Requests for the historical log file names on the TaskManager.- Returns:
- A Tuple2 Array with all log file names with its length.
-
sendOperatorEventToTask
CompletableFuture<Acknowledge> sendOperatorEventToTask(ExecutionAttemptID task, OperatorID operator, org.apache.flink.util.SerializedValue<OperatorEvent> evt)
Description copied from interface:TaskExecutorOperatorEventGateway
Sends an operator event to an operator in a task executed by the Task Manager (Task Executor).The reception is acknowledged (future is completed) when the event has been dispatched to the
AbstractInvokable.dispatchOperatorEvent(OperatorID, SerializedValue)
method. It is not guaranteed that the event is processed successfully within the implementation. These cases are up to the task and event sender to handle (for example with an explicit response message upon success, or by triggering failure/recovery upon exception).- Specified by:
sendOperatorEventToTask
in interfaceTaskExecutorOperatorEventGateway
-
requestThreadDump
CompletableFuture<ThreadDumpInfo> requestThreadDump(Duration timeout)
Requests the thread dump from this TaskManager.- Parameters:
timeout
- timeout for the asynchronous operation- Returns:
- the
ThreadDumpInfo
for this TaskManager.
-
updateDelegationTokens
CompletableFuture<Acknowledge> updateDelegationTokens(ResourceManagerId resourceManagerId, byte[] tokens)
Sends new delegation tokens to this TaskManager.- Parameters:
resourceManagerId
- current leader id of the ResourceManagertokens
- new tokens- Returns:
- Future acknowledge of the successful operation
-
getAndRetainPartitionWithMetrics
default CompletableFuture<Collection<PartitionWithMetrics>> getAndRetainPartitionWithMetrics(org.apache.flink.api.common.JobID jobId)
Get and retain all partitions and their metrics located on this task executor, the metrics mainly includes the meta information of partition(partition bytes, etc).- Parameters:
jobId
- ID of the target job- Returns:
- All partitions belong to the target job and their metrics
-
requestProfiling
CompletableFuture<ProfilingInfo> requestProfiling(int duration, ProfilingInfo.ProfilingMode mode, Duration timeout)
Requests the profiling from this TaskManager.- Parameters:
duration
- profiling durationmode
- profiling modeProfilingInfo.ProfilingMode
timeout
- timeout for the asynchronous operation- Returns:
- the
ProfilingInfo
for this TaskManager.
-
requestProfilingList
CompletableFuture<Collection<ProfilingInfo>> requestProfilingList(Duration timeout)
Requests for the historical profiling file names on the TaskManager.- Returns:
- A Collection with all profiling instances information.
-
-