org.apache.hadoop.yarn.client.api.async
Class AMRMClientAsync<T extends org.apache.hadoop.yarn.client.api.AMRMClient.ContainerRequest>

java.lang.Object
  extended by org.apache.hadoop.service.AbstractService
      extended by org.apache.hadoop.yarn.client.api.async.AMRMClientAsync<T>
All Implemented Interfaces:
Closeable, Service

@InterfaceAudience.Public
@InterfaceStability.Stable
public abstract class AMRMClientAsync<T extends org.apache.hadoop.yarn.client.api.AMRMClient.ContainerRequest>
extends AbstractService

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
<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)
           
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)
           
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

client

protected final AMRMClient<T extends org.apache.hadoop.yarn.client.api.AMRMClient.ContainerRequest> client

handler

protected final org.apache.hadoop.yarn.client.api.async.AMRMClientAsync.CallbackHandler handler

heartbeatIntervalMs

protected final AtomicInteger heartbeatIntervalMs
Constructor Detail

AMRMClientAsync

protected AMRMClientAsync(int intervalMs,
                          org.apache.hadoop.yarn.client.api.async.AMRMClientAsync.CallbackHandler callbackHandler)

AMRMClientAsync

@InterfaceAudience.Private
protected AMRMClientAsync(AMRMClient<T> client,
                                                    int intervalMs,
                                                    org.apache.hadoop.yarn.client.api.async.AMRMClientAsync.CallbackHandler callbackHandler)
Method Detail

createAMRMClientAsync

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)

createAMRMClientAsync

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)

setHeartbeatInterval

public void setHeartbeatInterval(int interval)

getMatchingRequests

public abstract List<? extends Collection<T>> getMatchingRequests(Priority priority,
                                                                  String resourceName,
                                                                  Resource capability)

registerApplicationMaster

public abstract RegisterApplicationMasterResponse registerApplicationMaster(String appHostName,
                                                                            int appHostPort,
                                                                            String appTrackingUrl)
                                                                     throws YarnException,
                                                                            IOException
Registers this application master with the resource manager. On successful registration, starts the heartbeating thread.

Throws:
YarnException
IOException

unregisterApplicationMaster

public abstract void unregisterApplicationMaster(FinalApplicationStatus appStatus,
                                                 String appMessage,
                                                 String appTrackingUrl)
                                          throws YarnException,
                                                 IOException
Unregister the application master. This must be called in the end.

Parameters:
appStatus - Success/Failure status of the master
appMessage - Diagnostics message on failure
appTrackingUrl - New URL to get master info
Throws:
YarnException
IOException

addContainerRequest

public abstract void addContainerRequest(T req)
Request containers for resources before calling allocate

Parameters:
req - Resource request

removeContainerRequest

public abstract void removeContainerRequest(T req)
Remove previous container request. The previous container request may have already been sent to the ResourceManager. So even after the remove request the app must be prepared to receive an allocation for the previous request even after the remove request

Parameters:
req - Resource request

releaseAssignedContainer

public abstract void releaseAssignedContainer(ContainerId containerId)
Release containers assigned by the Resource Manager. If the app cannot use the container or wants to give up the container then it can release them. The app needs to make new requests for the released resource capability if it still needs it. eg. it released non-local resources

Parameters:
containerId -

getAvailableResources

public abstract Resource getAvailableResources()
Get the currently available resources in the cluster. A valid value is available after a call to allocate has been made

Returns:
Currently available resources

getClusterNodeCount

public abstract int getClusterNodeCount()
Get the current number of nodes in the cluster. A valid values is available after a call to allocate has been made

Returns:
Current number of nodes in the cluster


Copyright © 2014 Apache Software Foundation. All Rights Reserved.