@InterfaceAudience.Public @InterfaceStability.Stable public abstract class AMRMClient<T extends org.apache.hadoop.yarn.client.api.AMRMClient.ContainerRequest> extends AbstractService
Modifier | Constructor and Description |
---|---|
protected |
AMRMClient(String name) |
Modifier and Type | Method and Description |
---|---|
abstract void |
addContainerRequest(T req)
Request containers for resources before calling
allocate |
void |
addSchedulingRequests(Collection<SchedulingRequest> schedulingRequests)
Add a Collection of SchedulingRequests.
|
abstract AllocateResponse |
allocate(float progressIndicator)
Request additional containers and receive new container allocations.
|
static <T extends org.apache.hadoop.yarn.client.api.AMRMClient.ContainerRequest> |
createAMRMClient()
Create a new instance of AMRMClient.
|
abstract Resource |
getAvailableResources()
Get the currently available resources in the cluster.
|
abstract int |
getClusterNodeCount()
Get the current number of nodes in the cluster.
|
abstract Collection<T> |
getMatchingRequests(long allocationRequestId)
Get outstanding
ContainerRequest s matching the given
allocationRequestId. |
List<? extends Collection<T>> |
getMatchingRequests(Priority priority,
String resourceName,
ExecutionType executionType,
Resource capability)
Get outstanding
ContainerRequest s matching the given
parameters. |
List<? extends Collection<T>> |
getMatchingRequests(Priority priority,
String resourceName,
ExecutionType executionType,
Resource capability,
String profile) |
abstract List<? extends Collection<T>> |
getMatchingRequests(Priority priority,
String resourceName,
Resource capability)
Get outstanding
ContainerRequest s matching the given
parameters. |
NMTokenCache |
getNMTokenCache()
Get the NM token cache of the
AMRMClient . |
org.apache.hadoop.yarn.client.api.TimelineV2Client |
getRegisteredTimelineV2Client()
Get registered timeline v2 client.
|
abstract RegisterApplicationMasterResponse |
registerApplicationMaster(String appHostName,
int appHostPort,
String appTrackingUrl)
Register the application master.
|
RegisterApplicationMasterResponse |
registerApplicationMaster(String appHostName,
int appHostPort,
String appTrackingUrl,
Map<Set<String>,PlacementConstraint> placementConstraints)
Register the application master.
|
void |
registerTimelineV2Client(org.apache.hadoop.yarn.client.api.TimelineV2Client client)
Register TimelineV2Client to AMRMClient.
|
abstract void |
releaseAssignedContainer(ContainerId containerId)
Release containers assigned by the Resource Manager.
|
abstract void |
removeContainerRequest(T req)
Remove previous container request.
|
void |
requestContainerResourceChange(Container container,
Resource capability)
Deprecated.
|
abstract void |
requestContainerUpdate(Container container,
UpdateContainerRequest updateContainerRequest)
Request a container update before calling
allocate . |
void |
setNMTokenCache(NMTokenCache nmTokenCache)
Set the NM token cache for the
AMRMClient . |
abstract void |
unregisterApplicationMaster(FinalApplicationStatus appStatus,
String appMessage,
String appTrackingUrl)
Unregister the application master.
|
abstract void |
updateBlacklist(List<String> blacklistAdditions,
List<String> blacklistRemovals)
Update application's blacklist with addition or removal resources.
|
void |
updateTrackingUrl(String trackingUrl)
Update application's tracking url on next heartbeat.
|
void |
waitFor(Supplier<Boolean> check)
Wait for
check to return true for each 1000 ms. |
void |
waitFor(Supplier<Boolean> check,
int checkEveryMillis)
Wait for
check to return true for each
checkEveryMillis ms. |
void |
waitFor(Supplier<Boolean> check,
int checkEveryMillis,
int logInterval)
Wait for
check to return true for each
checkEveryMillis ms. |
close, getBlockers, getConfig, getFailureCause, getFailureState, getLifecycleHistory, getName, getServiceState, getStartTime, init, isInState, noteFailure, putBlocker, registerGlobalListener, registerServiceListener, removeBlocker, serviceInit, serviceStart, serviceStop, setConfig, start, stop, toString, unregisterGlobalListener, unregisterServiceListener, waitForServiceToStop
@InterfaceAudience.Private protected AMRMClient(String name)
@InterfaceAudience.Public public static <T extends org.apache.hadoop.yarn.client.api.AMRMClient.ContainerRequest> AMRMClient<T> createAMRMClient()
AMRMClient.<T>createAMRMClientContainerRequest()
@InterfaceAudience.Public @InterfaceStability.Unstable public void addSchedulingRequests(Collection<SchedulingRequest> schedulingRequests)
schedulingRequests
- Collection of Scheduling Requests.public abstract RegisterApplicationMasterResponse registerApplicationMaster(String appHostName, int appHostPort, String appTrackingUrl) throws YarnException, IOException
appHostName
- Name of the host on which master is runningappHostPort
- Port master is listening onappTrackingUrl
- URL at which the master info can be seenRegisterApplicationMasterResponse
YarnException
IOException
@InterfaceAudience.Public @InterfaceStability.Unstable public RegisterApplicationMasterResponse registerApplicationMaster(String appHostName, int appHostPort, String appTrackingUrl, Map<Set<String>,PlacementConstraint> placementConstraints) throws YarnException, IOException
appHostName
- Name of the host on which master is runningappHostPort
- Port master is listening onappTrackingUrl
- URL at which the master info can be seenplacementConstraints
- Placement Constraints mappings.RegisterApplicationMasterResponse
YarnException
IOException
public abstract AllocateResponse allocate(float progressIndicator) throws YarnException, IOException
addContainerRequest
are sent to the
ResourceManager
. New containers assigned to the master are
retrieved. Status of completed containers and node health updates are also
retrieved. This also doubles up as a heartbeat to the ResourceManager and
must be made periodically. The call may not always return any new
allocations of containers. App should not make concurrent allocate
requests. May cause request loss.
Note : If the user has not removed container requests that have already been satisfied, then the re-register may end up sending the entire container requests to the RM (including matched requests). Which would mean the RM could end up giving it a lot of new allocated containers.
progressIndicator
- Indicates progress made by the masterYarnException
IOException
public abstract void unregisterApplicationMaster(FinalApplicationStatus appStatus, String appMessage, String appTrackingUrl) throws YarnException, IOException
appStatus
- Success/Failure status of the masterappMessage
- Diagnostics message on failureappTrackingUrl
- New URL to get master infoYarnException
IOException
public abstract void addContainerRequest(T req)
allocate
req
- Resource requestpublic abstract void removeContainerRequest(T req)
req
- Resource request@Deprecated public void requestContainerResourceChange(Container container, Resource capability)
requestContainerUpdate(Container, UpdateContainerRequest)
allocate
.
Any previous pending resource change request of the same container will be
removed.
Application that calls this method is expected to maintain the
Container
s that are returned from previous successful
allocations or resource changes. By passing in the existing container and a
target resource capability to this method, the application requests the
ResourceManager to change the existing resource allocation to the target
resource allocation.container
- The container returned from the last successful resource
allocation or resource changecapability
- The target resource capability of the containerpublic abstract void requestContainerUpdate(Container container, UpdateContainerRequest updateContainerRequest)
allocate
.
Any previous pending update request of the same container will be
removed.container
- The container returned from the last successful resource
allocation or updateupdateContainerRequest
- The UpdateContainerRequest
.public abstract void releaseAssignedContainer(ContainerId containerId)
containerId
- public abstract Resource getAvailableResources()
public abstract int getClusterNodeCount()
@InterfaceStability.Evolving public abstract List<? extends Collection<T>> getMatchingRequests(Priority priority, String resourceName, Resource capability)
ContainerRequest
s matching the given
parameters. These ContainerRequests should have been added via
addContainerRequest
earlier in the lifecycle. For performance,
the AMRMClient may return its internal collection directly without creating
a copy. Users should not perform mutable operations on the return value.
Each collection in the list contains requests with identical
Resource
size that fit in the given capability. In a
collection, requests will be returned in the same order as they were added.
NOTE: This API only matches Container requests that were created by the
client WITHOUT the allocationRequestId being set.@InterfaceStability.Evolving public List<? extends Collection<T>> getMatchingRequests(Priority priority, String resourceName, ExecutionType executionType, Resource capability)
ContainerRequest
s matching the given
parameters. These ContainerRequests should have been added via
addContainerRequest
earlier in the lifecycle. For performance,
the AMRMClient may return its internal collection directly without creating
a copy. Users should not perform mutable operations on the return value.
Each collection in the list contains requests with identical
Resource
size that fit in the given capability. In a
collection, requests will be returned in the same order as they were added.
specify an ExecutionType
.
NOTE: This API only matches Container requests that were created by the
client WITHOUT the allocationRequestId being set.priority
- PriorityresourceName
- LocationexecutionType
- ExecutionTypecapability
- Capability@InterfaceStability.Evolving public List<? extends Collection<T>> getMatchingRequests(Priority priority, String resourceName, ExecutionType executionType, Resource capability, String profile)
@InterfaceStability.Evolving public abstract Collection<T> getMatchingRequests(long allocationRequestId)
ContainerRequest
s matching the given
allocationRequestId. These ContainerRequests should have been added via
addContainerRequest
earlier in the lifecycle. For performance,
the AMRMClient may return its internal collection directly without creating
a copy. Users should not perform mutable operations on the return value.
NOTE: This API only matches Container requests that were created by the
client WITH the allocationRequestId being set to a non-default value.allocationRequestId
- Allocation Request Idpublic abstract void updateBlacklist(List<String> blacklistAdditions, List<String> blacklistRemovals)
blacklistAdditions
- list of resources which should be added to the
application blacklistblacklistRemovals
- list of resources which should be removed from the
application blacklistpublic void setNMTokenCache(NMTokenCache nmTokenCache)
AMRMClient
. This cache must
be shared with the NMClient
used to manage containers for the
AMRMClient
If a NM token cache is not set, the NMTokenCache.getSingleton()
singleton instance will be used.
nmTokenCache
- the NM token cache to use.public NMTokenCache getNMTokenCache()
AMRMClient
. This cache must be
shared with the NMClient
used to manage containers for the
AMRMClient
.
If a NM token cache is not set, the NMTokenCache.getSingleton()
singleton instance will be used.
public void registerTimelineV2Client(org.apache.hadoop.yarn.client.api.TimelineV2Client client)
client
- the timeline v2 client to registerpublic org.apache.hadoop.yarn.client.api.TimelineV2Client getRegisteredTimelineV2Client()
@InterfaceAudience.Public @InterfaceStability.Unstable public void updateTrackingUrl(String trackingUrl)
trackingUrl
- new tracking url for this applicationpublic void waitFor(Supplier<Boolean> check) throws InterruptedException
check
to return true for each 1000 ms.
See also waitFor(java.util.function.Supplier, int)
and waitFor(java.util.function.Supplier, int, int)
check
- the condition for which it should waitInterruptedException
public void waitFor(Supplier<Boolean> check, int checkEveryMillis) throws InterruptedException
check
to return true for each
checkEveryMillis
ms.
See also waitFor(java.util.function.Supplier, int, int)
check
- user defined checkercheckEveryMillis
- interval to call check
InterruptedException
public void waitFor(Supplier<Boolean> check, int checkEveryMillis, int logInterval) throws InterruptedException
check
to return true for each
checkEveryMillis
ms. In the main loop, this method will log
the message "waiting in main loop" for each logInterval
times
iteration to confirm the thread is alive.check
- user defined checkercheckEveryMillis
- interval to call check
logInterval
- interval to log for eachInterruptedException
Copyright © 2024 Apache Software Foundation. All rights reserved.