|
||||||||||
PREV CLASS NEXT CLASS | FRAMES NO FRAMES | |||||||||
SUMMARY: NESTED | FIELD | CONSTR | METHOD | DETAIL: FIELD | CONSTR | METHOD |
java.lang.Object org.apache.hadoop.service.AbstractService org.apache.hadoop.yarn.client.api.async.AMRMClientAsync<T>
@InterfaceAudience.Public @InterfaceStability.Stable public abstract class AMRMClientAsync<T extends org.apache.hadoop.yarn.client.api.AMRMClient.ContainerRequest>
AMRMClientAsync
handles communication with the ResourceManager
and provides asynchronous updates on events such as container allocations and
completions. It contains a thread that sends periodic heartbeats to the
ResourceManager.
It should be used by implementing a CallbackHandler:
class MyCallbackHandler implements AMRMClientAsync.CallbackHandler {
public void onContainersAllocated(List<Container> containers) {
[run tasks on the containers]
}
public void onContainersCompleted(List<ContainerStatus> statuses) {
[update progress, check whether app is done]
}
public void onNodesUpdated(List<NodeReport> updated) {}
public void onReboot() {}
}
The client's lifecycle should be managed similarly to the following:
AMRMClientAsync asyncClient =
createAMRMClientAsync(appAttId, 1000, new MyCallbackhandler());
asyncClient.init(conf);
asyncClient.start();
RegisterApplicationMasterResponse response = asyncClient
.registerApplicationMaster(appMasterHostname, appMasterRpcPort,
appMasterTrackingUrl);
asyncClient.addContainerRequest(containerRequest);
[... wait for application to complete]
asyncClient.unregisterApplicationMaster(status, appMsg, trackingUrl);
asyncClient.stop();
Field Summary | |
---|---|
protected AMRMClient<T> |
client
|
protected org.apache.hadoop.yarn.client.api.async.AMRMClientAsync.CallbackHandler |
handler
|
protected AtomicInteger |
heartbeatIntervalMs
|
Constructor Summary | |
---|---|
protected |
AMRMClientAsync(AMRMClient<T> client,
int intervalMs,
org.apache.hadoop.yarn.client.api.async.AMRMClientAsync.CallbackHandler callbackHandler)
|
protected |
AMRMClientAsync(int intervalMs,
org.apache.hadoop.yarn.client.api.async.AMRMClientAsync.CallbackHandler callbackHandler)
|
Method Summary | ||
---|---|---|
abstract void |
addContainerRequest(T req)
Request containers for resources before calling allocate |
|
static
|
createAMRMClientAsync(AMRMClient<T> client,
int intervalMs,
org.apache.hadoop.yarn.client.api.async.AMRMClientAsync.CallbackHandler callbackHandler)
|
|
static
|
createAMRMClientAsync(int intervalMs,
org.apache.hadoop.yarn.client.api.async.AMRMClientAsync.CallbackHandler callbackHandler)
|
|
abstract Resource |
getAvailableResources()
Get the currently available resources in the cluster. |
|
abstract int |
getClusterNodeCount()
Get the current number of nodes in the cluster. |
|
abstract List<? extends Collection<T>> |
getMatchingRequests(Priority priority,
String resourceName,
Resource capability)
|
|
abstract RegisterApplicationMasterResponse |
registerApplicationMaster(String appHostName,
int appHostPort,
String appTrackingUrl)
Registers this application master with the resource manager. |
|
abstract void |
releaseAssignedContainer(ContainerId containerId)
Release containers assigned by the Resource Manager. |
|
abstract void |
removeContainerRequest(T req)
Remove previous container request. |
|
void |
setHeartbeatInterval(int interval)
|
|
abstract void |
unregisterApplicationMaster(FinalApplicationStatus appStatus,
String appMessage,
String appTrackingUrl)
Unregister the application master. |
Methods inherited from class org.apache.hadoop.service.AbstractService |
---|
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 |
Methods inherited from class java.lang.Object |
---|
clone, equals, finalize, getClass, hashCode, notify, notifyAll, wait, wait, wait |
Field Detail |
---|
protected final AMRMClient<T extends org.apache.hadoop.yarn.client.api.AMRMClient.ContainerRequest> client
protected final org.apache.hadoop.yarn.client.api.async.AMRMClientAsync.CallbackHandler handler
protected final AtomicInteger heartbeatIntervalMs
Constructor Detail |
---|
protected AMRMClientAsync(int intervalMs, org.apache.hadoop.yarn.client.api.async.AMRMClientAsync.CallbackHandler callbackHandler)
@InterfaceAudience.Private protected AMRMClientAsync(AMRMClient<T> client, int intervalMs, org.apache.hadoop.yarn.client.api.async.AMRMClientAsync.CallbackHandler callbackHandler)
Method Detail |
---|
public static <T extends org.apache.hadoop.yarn.client.api.AMRMClient.ContainerRequest> AMRMClientAsync<T> createAMRMClientAsync(int intervalMs, org.apache.hadoop.yarn.client.api.async.AMRMClientAsync.CallbackHandler callbackHandler)
public static <T extends org.apache.hadoop.yarn.client.api.AMRMClient.ContainerRequest> AMRMClientAsync<T> createAMRMClientAsync(AMRMClient<T> client, int intervalMs, org.apache.hadoop.yarn.client.api.async.AMRMClientAsync.CallbackHandler callbackHandler)
public void setHeartbeatInterval(int interval)
public abstract List<? extends Collection<T>> getMatchingRequests(Priority priority, String resourceName, Resource capability)
public abstract RegisterApplicationMasterResponse registerApplicationMaster(String appHostName, int appHostPort, String appTrackingUrl) throws YarnException, IOException
YarnException
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 info
YarnException
IOException
public abstract void addContainerRequest(T req)
allocate
req
- Resource requestpublic abstract void removeContainerRequest(T req)
req
- Resource requestpublic abstract void releaseAssignedContainer(ContainerId containerId)
containerId
- public abstract Resource getAvailableResources()
public abstract int getClusterNodeCount()
|
||||||||||
PREV CLASS NEXT CLASS | FRAMES NO FRAMES | |||||||||
SUMMARY: NESTED | FIELD | CONSTR | METHOD | DETAIL: FIELD | CONSTR | METHOD |