org.apache.hadoop.yarn.client.api
Class AMRMClient<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.AMRMClient<T>
All Implemented Interfaces:
Closeable, Service

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


Constructor Summary
protected AMRMClient(String name)
           
 
Method Summary
abstract  void addContainerRequest(T req)
          Request containers for resources before calling allocate
abstract  AllocateResponse allocate(float progressIndicator)
          Request additional containers and receive new container allocations.
static
<T extends org.apache.hadoop.yarn.client.api.AMRMClient.ContainerRequest>
AMRMClient<T>
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  List<? extends Collection<T>> getMatchingRequests(Priority priority, String resourceName, Resource capability)
          Get outstanding ContainerRequests matching the given parameters.
 NMTokenCache getNMTokenCache()
          Get the NM token cache of the AMRMClient.
abstract  RegisterApplicationMasterResponse registerApplicationMaster(String appHostName, int appHostPort, String appTrackingUrl)
          Register the application master.
abstract  void releaseAssignedContainer(ContainerId containerId)
          Release containers assigned by the Resource Manager.
abstract  void removeContainerRequest(T req)
          Remove previous container request.
 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.
 
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
 

Constructor Detail

AMRMClient

@InterfaceAudience.Private
protected AMRMClient(String name)
Method Detail

createAMRMClient

@InterfaceAudience.Public
public static <T extends org.apache.hadoop.yarn.client.api.AMRMClient.ContainerRequest> AMRMClient<T> createAMRMClient()
Create a new instance of AMRMClient. For usage:
 AMRMClient.<T>createAMRMClientContainerRequest()
 

Returns:
the newly create AMRMClient instance.

registerApplicationMaster

public abstract RegisterApplicationMasterResponse registerApplicationMaster(String appHostName,
                                                                            int appHostPort,
                                                                            String appTrackingUrl)
                                                                     throws YarnException,
                                                                            IOException
Register the application master. This must be called before any other interaction

Parameters:
appHostName - Name of the host on which master is running
appHostPort - Port master is listening on
appTrackingUrl - URL at which the master info can be seen
Returns:
RegisterApplicationMasterResponse
Throws:
YarnException
IOException

allocate

public abstract AllocateResponse allocate(float progressIndicator)
                                   throws YarnException,
                                          IOException
Request additional containers and receive new container allocations. Requests made via 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.

Parameters:
progressIndicator - Indicates progress made by the master
Returns:
the response of the allocate request
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

getMatchingRequests

public abstract List<? extends Collection<T>> getMatchingRequests(Priority priority,
                                                                  String resourceName,
                                                                  Resource capability)
Get outstanding ContainerRequests 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.

Returns:
Collection of request matching the parameters

updateBlacklist

public abstract void updateBlacklist(List<String> blacklistAdditions,
                                     List<String> blacklistRemovals)
Update application's blacklist with addition or removal resources.

Parameters:
blacklistAdditions - list of resources which should be added to the application blacklist
blacklistRemovals - list of resources which should be removed from the application blacklist

setNMTokenCache

public void setNMTokenCache(NMTokenCache nmTokenCache)
Set the NM token cache for the 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.

Parameters:
nmTokenCache - the NM token cache to use.

getNMTokenCache

public NMTokenCache getNMTokenCache()
Get the NM token cache of the 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.

Returns:
the NM token cache.


Copyright © 2014 Apache Software Foundation. All Rights Reserved.