public interface ResourceManagerGateway extends FencedRpcGateway<ResourceManagerId>
ResourceManager
's RPC gateway interface.Modifier and Type | Method and Description |
---|---|
void |
cancelSlotRequest(AllocationID allocationID)
Cancel the slot allocation requests from the resource manager.
|
CompletableFuture<Acknowledge> |
deregisterApplication(ApplicationStatus finalStatus,
String diagnostics)
Deregister Flink from the underlying resource management system.
|
void |
disconnectJobManager(JobID jobId,
Exception cause)
Disconnects a JobManager specified by the given resourceID from the
ResourceManager . |
void |
disconnectTaskManager(ResourceID resourceID,
Exception cause)
Disconnects a TaskManager specified by the given resourceID from the
ResourceManager . |
CompletableFuture<Integer> |
getNumberOfRegisteredTaskManagers()
Gets the currently registered number of TaskManagers.
|
void |
heartbeatFromJobManager(ResourceID heartbeatOrigin)
Sends the heartbeat to resource manager from job manager
|
void |
heartbeatFromTaskManager(ResourceID heartbeatOrigin,
SlotReport slotReport)
Sends the heartbeat to resource manager from task manager
|
void |
notifySlotAvailable(InstanceID instanceId,
SlotID slotID,
AllocationID oldAllocationId)
Sent by the TaskExecutor to notify the ResourceManager that a slot has become available.
|
void |
registerInfoMessageListener(String infoMessageListenerAddress)
Registers an infoMessage listener
|
CompletableFuture<RegistrationResponse> |
registerJobManager(JobMasterId jobMasterId,
ResourceID jobMasterResourceId,
String jobMasterAddress,
JobID jobId,
Time timeout)
Register a
JobMaster at the resource manager. |
CompletableFuture<RegistrationResponse> |
registerTaskExecutor(String taskExecutorAddress,
ResourceID resourceId,
int dataPort,
HardwareDescription hardwareDescription,
Time timeout)
Register a
TaskExecutor at the resource manager. |
CompletableFuture<ResourceOverview> |
requestResourceOverview(Time timeout)
Requests the resource overview.
|
CompletableFuture<Acknowledge> |
requestSlot(JobMasterId jobMasterId,
SlotRequest slotRequest,
Time timeout)
Requests a slot from the resource manager.
|
CompletableFuture<TransientBlobKey> |
requestTaskManagerFileUpload(ResourceID taskManagerId,
FileType fileType,
Time timeout)
Request the file upload from the given
TaskExecutor to the cluster's BlobServer . |
CompletableFuture<TaskManagerInfo> |
requestTaskManagerInfo(ResourceID taskManagerId,
Time timeout)
Requests information about the given
TaskExecutor . |
CompletableFuture<Collection<TaskManagerInfo>> |
requestTaskManagerInfo(Time timeout)
Requests information about the registered
TaskExecutor . |
CompletableFuture<Collection<Tuple2<ResourceID,String>>> |
requestTaskManagerMetricQueryServicePaths(Time timeout)
Requests the paths for the TaskManager's
MetricQueryService to query. |
CompletableFuture<Acknowledge> |
sendSlotReport(ResourceID taskManagerResourceId,
InstanceID taskManagerRegistrationId,
SlotReport slotReport,
Time timeout)
Sends the given
SlotReport to the ResourceManager. |
void |
unRegisterInfoMessageListener(String infoMessageListenerAddress)
Unregisters an infoMessage listener
|
getFencingToken
getAddress, getHostname
CompletableFuture<RegistrationResponse> registerJobManager(JobMasterId jobMasterId, ResourceID jobMasterResourceId, String jobMasterAddress, JobID jobId, Time timeout)
JobMaster
at the resource manager.jobMasterId
- The fencing token for the JobMaster leaderjobMasterResourceId
- The resource ID of the JobMaster that registersjobMasterAddress
- The address of the JobMaster that registersjobId
- The Job ID of the JobMaster that registerstimeout
- Timeout for the future to completeCompletableFuture<Acknowledge> requestSlot(JobMasterId jobMasterId, SlotRequest slotRequest, Time timeout)
jobMasterId
- id of the JobMasterslotRequest
- The slot to requestvoid cancelSlotRequest(AllocationID allocationID)
allocationID
- The slot to requestCompletableFuture<RegistrationResponse> registerTaskExecutor(String taskExecutorAddress, ResourceID resourceId, int dataPort, HardwareDescription hardwareDescription, Time timeout)
TaskExecutor
at the resource manager.taskExecutorAddress
- The address of the TaskExecutor that registersresourceId
- The resource ID of the TaskExecutor that registersdataPort
- port used for data communication between TaskExecutorshardwareDescription
- of the registering TaskExecutortimeout
- The timeout for the response.CompletableFuture<Acknowledge> sendSlotReport(ResourceID taskManagerResourceId, InstanceID taskManagerRegistrationId, SlotReport slotReport, Time timeout)
SlotReport
to the ResourceManager.taskManagerRegistrationId
- id identifying the sending TaskManagerslotReport
- which is sent to the ResourceManagertimeout
- for the operationAcknowledge
once the slot report has been received.void notifySlotAvailable(InstanceID instanceId, SlotID slotID, AllocationID oldAllocationId)
instanceId
- TaskExecutor's instance idslotID
- The SlotID of the freed slotoldAllocationId
- to which the slot has been allocatedvoid registerInfoMessageListener(String infoMessageListenerAddress)
infoMessageListenerAddress
- address of infoMessage listener to register to this resource managervoid unRegisterInfoMessageListener(String infoMessageListenerAddress)
infoMessageListenerAddress
- address of infoMessage listener to unregister from this resource managerCompletableFuture<Acknowledge> deregisterApplication(ApplicationStatus finalStatus, @Nullable String diagnostics)
finalStatus
- final status with which to deregister the Flink applicationdiagnostics
- additional information for the resource management system, can be null
CompletableFuture<Integer> getNumberOfRegisteredTaskManagers()
void heartbeatFromTaskManager(ResourceID heartbeatOrigin, SlotReport slotReport)
heartbeatOrigin
- unique id of the task managerslotReport
- Current slot allocation on the originating TaskManagervoid heartbeatFromJobManager(ResourceID heartbeatOrigin)
heartbeatOrigin
- unique id of the job managervoid disconnectTaskManager(ResourceID resourceID, Exception cause)
ResourceManager
.resourceID
- identifying the TaskManager to disconnectcause
- for the disconnection of the TaskManagervoid disconnectJobManager(JobID jobId, Exception cause)
ResourceManager
.jobId
- JobID for which the JobManager was the leadercause
- for the disconnection of the JobManagerCompletableFuture<Collection<TaskManagerInfo>> requestTaskManagerInfo(Time timeout)
TaskExecutor
.timeout
- of the requestCompletableFuture<TaskManagerInfo> requestTaskManagerInfo(ResourceID taskManagerId, Time timeout)
TaskExecutor
.taskManagerId
- identifying the TaskExecutor for which to return informationtimeout
- of the requestCompletableFuture<ResourceOverview> requestResourceOverview(Time timeout)
timeout
- of the requestCompletableFuture<Collection<Tuple2<ResourceID,String>>> requestTaskManagerMetricQueryServicePaths(Time timeout)
MetricQueryService
to query.timeout
- for the asynchronous operationCompletableFuture<TransientBlobKey> requestTaskManagerFileUpload(ResourceID taskManagerId, FileType fileType, Time timeout)
TaskExecutor
to the cluster's BlobServer
. The
corresponding TransientBlobKey
is returned.taskManagerId
- identifying the TaskExecutor
to upload the specified filefileType
- type of the file to uploadtimeout
- for the asynchronous operationTransientBlobKey
after uploading the file to the
BlobServer
.Copyright © 2014–2020 The Apache Software Foundation. All rights reserved.