12345678910111213141516171819202122232425262728293031323334353637383940414243444546474849505152535455565758596061626364656667686970717273747576777879808182838485868788899091929394959697989910010110210310410510610710810911011111211311411511611711811912012112212312412512612712812913013113213313413513613713813914014114214314414514614714814915015115215315415515615715815916016116216316416516616716816917017117217317417517617717817918018118218318418518618718818919019119219319419519619719819920020120220320420520620720820921021121221321421521621721821922022122222322422522622722822923023123223323423523623723823924024124224324424524624724824925025125225325425525625725825926026126226326426526626726826927027127227327427527627727827928028128228328428528628728828929029129229329429529629729829930030130230330430530630730830931031131231331431531631731831932032132232332432532632732832933033133233333433533633733833934034134234334434534634734834935035135235335435535635735835936036136236336436536636736836937037137237337437537637737837938038138238338438538638738838939039139239339439539639739839940040140240340440540640740840941041141241341441541641741841942042142242342442542642742842943043143243343443543643743843944044144244344444544644744844945045145245345445545645745845946046146246346446546646746846947047147247347447547647747847948048148248348448548648748848949049149249349449549649749849950050150250350450550650750850951051151251351451551651751851952052152252352452552652752852953053153253353453553653753853954054154254354454554654754854955055155255355455555655755855956056156256356456556656756856957057157257357457557657757857958058158258358458558658758858959059159259359459559659759859960060160260360460560660760860961061161261361461561661761861962062162262362462562662762862963063163263363463563663763863964064164264364464564664764864965065165265365465565665765865966066166266366466566666766866967067167267367467567667767867968068168268368468568668768868969069169269369469569669769869970070170270370470570670770870971071171271371471571671771871972072172272372472572672772872973073173273373473573673773873974074174274374474574674774874975075175275375475575675775875976076176276376476576676776876977077177277377477577677777877978078178278378478578678778878979079179279379479579679779879980080180280380480580680780880981081181281381481581681781881982082182282382482582682782882983083183283383483583683783883984084184284384484584684784884985085185285385485585685785885986086186286386486586686786886987087187287387487587687787887988088188288388488588688788888989089189289389489589689789889990090190290390490590690790890991091191291391491591691791891992092192292392492592692792892993093193293393493593693793893994094194294394494594694794894995095195295395495595695795895996096196296396496596696796896997097197297397497597697797897998098198298398498598698798898999099199299399499599699799899910001001100210031004100510061007100810091010101110121013101410151016101710181019102010211022102310241025102610271028102910301031103210331034103510361037103810391040104110421043104410451046104710481049105010511052105310541055105610571058105910601061106210631064106510661067106810691070107110721073107410751076107710781079108010811082108310841085108610871088108910901091109210931094109510961097109810991100110111021103110411051106110711081109111011111112111311141115111611171118111911201121112211231124112511261127112811291130113111321133113411351136113711381139114011411142114311441145114611471148114911501151115211531154115511561157115811591160116111621163116411651166116711681169117011711172117311741175117611771178117911801181118211831184118511861187118811891190119111921193119411951196119711981199120012011202120312041205120612071208120912101211121212131214121512161217121812191220122112221223122412251226122712281229123012311232123312341235123612371238123912401241124212431244124512461247124812491250125112521253125412551256125712581259126012611262126312641265126612671268126912701271127212731274127512761277127812791280128112821283128412851286128712881289129012911292129312941295129612971298129913001301130213031304130513061307130813091310131113121313131413151316131713181319132013211322132313241325132613271328132913301331133213331334133513361337133813391340134113421343134413451346134713481349135013511352135313541355135613571358135913601361136213631364136513661367136813691370137113721373137413751376137713781379138013811382138313841385138613871388138913901391139213931394139513961397139813991400140114021403140414051406140714081409141014111412141314141415141614171418141914201421142214231424142514261427142814291430143114321433143414351436143714381439144014411442144314441445144614471448144914501451145214531454145514561457145814591460146114621463146414651466146714681469 |
- ~~ Licensed under the Apache License, Version 2.0 (the "License");
- ~~ you may not use this file except in compliance with the License.
- ~~ You may obtain a copy of the License at
- ~~
- ~~ http://www.apache.org/licenses/LICENSE-2.0
- ~~
- ~~ Unless required by applicable law or agreed to in writing, software
- ~~ distributed under the License is distributed on an "AS IS" BASIS,
- ~~ WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
- ~~ See the License for the specific language governing permissions and
- ~~ limitations under the License. See accompanying LICENSE file.
- ---
- ResourceManager REST API's.
- ---
- ---
- ${maven.build.timestamp}
- ResourceManager REST API's.
- \[ {{{./index.html}Go Back}} \]
- %{toc|section=1|fromDepth=0|toDepth=2}
- * Overview
- The ResourceManager REST API's allow the user to get information about the cluster - status on the cluster, metrics on the cluster, scheduler information, information about nodes in the cluster, and information about applications on the cluster.
-
- * Cluster Information API
- The cluster information resource provides overall information about the cluster.
- ** URI
- Both of the following URI's give you the cluster information.
- ------
- * http://<rm http address:port>/ws/v1/cluster
- * http://<rm http address:port>/ws/v1/cluster/info
- ------
- ** HTTP Operations Supported
- ------
- * GET
- ------
- ** Query Parameters Supported
- ------
- None
- ------
- ** Elements of the <clusterInfo> object
- *---------------+--------------+-------------------------------+
- || Item || Data Type || Description |
- *---------------+--------------+-------------------------------+
- | id | long | The cluster id |
- *---------------+--------------+-------------------------------+
- | startedOn | long | The time the cluster started (in ms since epoch)|
- *---------------+--------------+-------------------------------+
- | state | string | The ResourceManager state - valid values are: NOTINITED, INITED, STARTED, STOPPED|
- *---------------+--------------+-------------------------------+
- | resourceManagerVersion | string | Version of the ResourceManager |
- *---------------+--------------+-------------------------------+
- | resourceManagerBuildVersion | string | ResourceManager build string with build version, user, and checksum |
- *---------------+--------------+-------------------------------+
- | resourceManagerVersionBuiltOn | string | Timestamp when ResourceManager was built (in ms since epoch)|
- *---------------+--------------+-------------------------------+
- | hadoopVersion | string | Version of hadoop common |
- *---------------+--------------+-------------------------------+
- | hadoopBuildVersion | string | Hadoop common build string with build version, user, and checksum |
- *---------------+--------------+-------------------------------+
- | hadoopVersionBuiltOn | string | Timestamp when hadoop common was built(in ms since epoch)|
- *---------------+--------------+-------------------------------+
- ** Response Examples
- <<JSON response>>
- HTTP Request:
- ------
- GET http://<rm http address:port>/ws/v1/cluster/info
- ------
- Response Header:
- +---+
- HTTP/1.1 200 OK
- Content-Type: application/json
- Transfer-Encoding: chunked
- Server: Jetty(6.1.26)
- +---+
- Response Body:
- +---+
- {
- "clusterInfo":
- {
- "id":1324053971963,
- "startedOn":1324053971963,
- "state":"STARTED",
- "resourceManagerVersion":"0.23.1-SNAPSHOT",
- "resourceManagerBuildVersion":"0.23.1-SNAPSHOT from 1214049 by user1 source checksum 050cd664439d931c8743a6428fd6a693",
- "resourceManagerVersionBuiltOn":"Tue Dec 13 22:12:48 CST 2011",
- "hadoopVersion":"0.23.1-SNAPSHOT",
- "hadoopBuildVersion":"0.23.1-SNAPSHOT from 1214049 by user1 source checksum 11458df3bb77342dca5f917198fad328",
- "hadoopVersionBuiltOn":"Tue Dec 13 22:12:26 CST 2011"
- }
- }
- +---+
- <<XML response>>
- HTTP Request:
- -----
- Accept: application/xml
- GET http://<rm http address:port>/ws/v1/cluster/info
- -----
- Response Header:
- +---+
- HTTP/1.1 200 OK
- Content-Type: application/xml
- Content-Length: 712
- Server: Jetty(6.1.26)
- +---+
- Response Body:
- +---+
- <?xml version="1.0" encoding="UTF-8" standalone="yes"?>
- <clusterInfo>
- <id>1324053971963</id>
- <startedOn>1324053971963</startedOn>
- <state>STARTED</state>
- <resourceManagerVersion>0.23.1-SNAPSHOT</resourceManagerVersion>
- <resourceManagerBuildVersion>0.23.1-SNAPSHOT from 1214049 by user1 source checksum 050cd664439d931c8743a6428fd6a693</resourceManagerBuildVersion>
- <resourceManagerVersionBuiltOn>Tue Dec 13 22:12:48 CST 2011</resourceManagerVersionBuiltOn>
- <hadoopVersion>0.23.1-SNAPSHOT</hadoopVersion>
- <hadoopBuildVersion>0.23.1-SNAPSHOT from 1214049 by user1 source checksum 11458df3bb77342dca5f917198fad328</hadoopBuildVersion>
- <hadoopVersionBuiltOn>Tue Dec 13 22:12:48 CST 2011</hadoopVersionBuiltOn>
- </clusterInfo>
- +---+
- * Cluster Metrics API
- The cluster metrics resource provides some overall metrics about the cluster. More detailed metrics should be retrieved from the jmx interface.
- ** URI
- ------
- * http://<rm http address:port>/ws/v1/cluster/metrics
- ------
- ** HTTP Operations Supported
- ------
- * GET
- ------
- ** Query Parameters Supported
- ------
- None
- ------
- ** Elements of the <clusterMetrics> object
- *---------------+--------------+-------------------------------+
- || Item || Data Type || Description |
- *---------------+--------------+-------------------------------+
- | appsSubmitted | int | The number of applications submitted |
- *---------------+--------------+-------------------------------+
- | reservedMB | long | The amount of memory reserved in MB |
- *---------------+--------------+-------------------------------+
- | availableMB | long | The amount of memory available in MB |
- *---------------+--------------+-------------------------------+
- | allocatedMB | long | The amount of memory allocated in MB |
- *---------------+--------------+-------------------------------+
- | totalMB | long | The amount of total memory in MB |
- *---------------+--------------+-------------------------------+
- | containersAllocated | int | The number of containers allocated |
- *---------------+--------------+-------------------------------+
- | totalNodes | int | The total number of nodes |
- *---------------+--------------+-------------------------------+
- | activeNodes | int | The number of active nodes |
- *---------------+--------------+-------------------------------+
- | lostNodes | int | The number of lost nodes |
- *---------------+--------------+-------------------------------+
- | unhealthyNodes | int | The number of unhealthy nodes |
- *---------------+--------------+-------------------------------+
- | decommissionedNodes | int | The number of nodes decommissioned |
- *---------------+--------------+-------------------------------+
- | rebootedNodes | int | The number of nodes rebooted |
- *---------------+--------------+-------------------------------+
- ** Response Examples
- <<JSON response>>
- HTTP Request:
- ------
- GET http://<rm http address:port>/ws/v1/cluster/metrics
- ------
- Response Header:
- +---+
- HTTP/1.1 200 OK
- Content-Type: application/json
- Transfer-Encoding: chunked
- Server: Jetty(6.1.26)
- +---+
- Response Body:
- +---+
- {
- "clusterMetrics":
- {
- "appsSubmitted":4,
- "reservedMB":0,
- "availableMB":8192,
- "allocatedMB":0,
- "totalMB":8192,
- "containersAllocated":0,
- "totalNodes":1,
- "activeNodes":1,
- "lostNodes":0,
- "unhealthyNodes":0,
- "decommissionedNodes":0,
- "rebootedNodes":0
- }
- }
- +---+
- <<XML response>>
- HTTP Request:
- ------
- GET http://<rm http address:port>/ws/v1/cluster/metrics
- Accept: application/xml
- ------
- Response Header:
- +---+
- HTTP/1.1 200 OK
- Content-Type: application/xml
- Content-Length: 432
- Server: Jetty(6.1.26)
- +---+
- Response Body:
- +---+
- <?xml version="1.0" encoding="UTF-8" standalone="yes"?>
- <clusterMetrics>
- <appsSubmitted>4</appsSubmitted>
- <reservedMB>0</reservedMB>
- <availableMB>8192</availableMB>
- <allocatedMB>0</allocatedMB>
- <containersAllocated>0</containersAllocated>
- <totalMB>8192</totalMB>
- <totalNodes>1</totalNodes>
- <activeNodes>1</activeNodes>
- <lostNodes>0</lostNodes>
- <unhealthyNodes>0</unhealthyNodes>
- <decommissionedNodes>0</decommissionedNodes>
- <rebootedNodes>0</rebootedNodes>
- </clusterMetrics>
- +---+
- * Cluster Scheduler API
- A scheduler resource contains information about the current scheduler configured in a cluster. It currently supports both the Fifo and Capacity Scheduler. You will get different information depending on which scheduler is configured so be sure to look at the type information.
- ** URI
- ------
- * http://<rm http address:port>/ws/v1/cluster/scheduler
- ------
- ** HTTP Operations Supported
- ------
- * GET
- ------
- ** Query Parameters Supported
- ------
- None
- ------
- ** Capacity Scheduler API
- The capacity scheduler supports hierarchical queues. This one request will print information about all the queues and any subqueues they have.
- Queues that can actually have jobs submitted to them are referred to as leaf queues. These queues have additional data associated with them.
- ** Elements of the <schedulerInfo> object
- *---------------+--------------+-------------------------------+
- || Item || Data Type || Description |
- *---------------+--------------+-------------------------------+
- | type | string | Scheduler type - capacityScheduler|
- *---------------+--------------+-------------------------------+
- | capacity | float | Queue capacity in percentage relative to its parent queue |
- *---------------+--------------+-------------------------------+
- | usedCapacity | float | Used queue capacity in percentage relative its to parent queue |
- *---------------+--------------+-------------------------------+
- | maxCapacity | float | Maximum queue capacity in percentage relative to its parent queue |
- *---------------+--------------+-------------------------------+
- | queueName | string | Name of the queue |
- *---------------+--------------+-------------------------------+
- | queues | array of queues(JSON)/zero or more queue objects(XML) | A collection of queue resources|
- *---------------+--------------+-------------------------------+
- ** Elements of the queues/subQueues object for a Parent queue
- *---------------+--------------+-------------------------------+
- || Item || Data Type || Description |
- *---------------+--------------+-------------------------------+
- | capacity | float | Queue capacity in percentage relative to its parent queue |
- *---------------+--------------+-------------------------------+
- | usedCapacity | float | Used queue capacity in percentage relative its to parent queue |
- *---------------+--------------+-------------------------------+
- | maxCapacity | float | Maximum queue capacity in percentage relative to its parent queue |
- *---------------+--------------+-------------------------------+
- | absoluteCapacity | float | Absolute capacity percentage this queue can use of entire cluster |
- *---------------+--------------+-------------------------------+
- | absoluteMaxCapacity | float | Absolute maximum capacity percentage this queue can use of the entire cluster |
- *---------------+--------------+-------------------------------+
- | utilization | float | Queue utilization percentage relative to the entire cluster |
- *---------------+--------------+-------------------------------+
- | numApplications | int | The number of applications currently in the queue |
- *---------------+--------------+-------------------------------+
- | usedResources | string | A string describing the current resources used by the queue |
- *---------------+--------------+-------------------------------+
- | queueName | string | The name of the queue |
- *---------------+--------------+-------------------------------+
- | state | string of QueueState | The state of the queue |
- *---------------+--------------+-------------------------------+
- | subQueues | array of queues(JSON)/zero or more queue objects(XML) | A collection of sub-queue information|
- *---------------+--------------+-------------------------------+
- ** Elements of the queues/subQueues object for a Leaf queue - contains all elements in parent plus the following:
- *---------------+--------------+-------------------------------+
- || Item || Data Type || Description |
- *---------------+--------------+-------------------------------+
- | type | String | type of the queue - capacitySchedulerLeafQueueInfo |
- *---------------+--------------+-------------------------------+
- | numActiveApplications | int | The number of active applications in this queue |
- *---------------+--------------+-------------------------------+
- | numPendingApplications | int | The number of pending applications in this queue |
- *---------------+--------------+-------------------------------+
- | numContainers | int | The number of containers being used |
- *---------------+--------------+-------------------------------+
- | maxApplications | int | The maximum number of applications this queue can have |
- *---------------+--------------+-------------------------------+
- | maxApplicationsPerUser | int | The maximum number of applications per user this queue can have |
- *---------------+--------------+-------------------------------+
- | maxActiveApplications | int | The maximum number of active applications this queue can have |
- *---------------+--------------+-------------------------------+
- | maxActiveApplicationsPerUser | int | The maximum number of active applications per user this queue can have|
- *---------------+--------------+-------------------------------+
- | userLimit | int | The minimum user limit percent set in the configuration |
- *---------------+--------------+-------------------------------+
- | userLimitFactor | float | The user limit factor set in the configuration |
- *---------------+--------------+-------------------------------+
- *** Response Examples
- <<JSON response>>
- HTTP Request:
- ------
- GET http://<rm http address:port>/ws/v1/cluster/scheduler
- ------
- Response Header:
- +---+
- HTTP/1.1 200 OK
- Content-Type: application/json
- Transfer-Encoding: chunked
- Server: Jetty(6.1.26)
- +---+
- Response Body:
- +---+
- {
- "scheduler" : {
- "schedulerInfo" : {
- "queueName" : "root",
- "maxCapacity" : 100,
- "type" : "capacityScheduler",
- "queues" : [
- {
- "numPendingApplications" : 0,
- "queueName" : "default",
- "userLimitFactor" : 1,
- "maxApplications" : 7000,
- "usedCapacity" : 0,
- "numContainers" : 0,
- "state" : "RUNNING",
- "maxCapacity" : 90,
- "numApplications" : 0,
- "usedResources" : "memory: 0",
- "absoluteMaxCapacity" : 90,
- "maxActiveApplications" : 1,
- "numActiveApplications" : 0,
- "utilization" : 0,
- "userLimit" : 100,
- "absoluteCapacity" : 70,
- "maxActiveApplicationsPerUser" : 1,
- "capacity" : 70,
- "type" : "capacitySchedulerLeafQueueInfo",
- "maxApplicationsPerUser" : 7000
- },
- {
- "queueName" : "test",
- "utilization" : 0,
- "absoluteCapacity" : 20,
- "usedCapacity" : 0,
- "capacity" : 20,
- "subQueues" : [
- {
- "numPendingApplications" : 0,
- "queueName" : "a1",
- "userLimitFactor" : 1,
- "maxApplications" : 1200,
- "usedCapacity" : 0,
- "numContainers" : 0,
- "state" : "RUNNING",
- "maxCapacity" : 80,
- "numApplications" : 0,
- "usedResources" : "memory: 0",
- "absoluteMaxCapacity" : 16.000002,
- "maxActiveApplications" : 1,
- "numActiveApplications" : 0,
- "utilization" : 0,
- "userLimit" : 100,
- "absoluteCapacity" : 12,
- "maxActiveApplicationsPerUser" : 1,
- "capacity" : 60.000004,
- "type" : "capacitySchedulerLeafQueueInfo",
- "maxApplicationsPerUser" : 1200
- },
- {
- "numPendingApplications" : 0,
- "queueName" : "a2",
- "userLimitFactor" : 1,
- "maxApplications" : 800,
- "usedCapacity" : 0,
- "numContainers" : 0,
- "state" : "RUNNING",
- "maxCapacity" : 100,
- "numApplications" : 0,
- "usedResources" : "memory: 0",
- "absoluteMaxCapacity" : 100,
- "maxActiveApplications" : 1,
- "numActiveApplications" : 0,
- "utilization" : 0,
- "userLimit" : 100,
- "absoluteCapacity" : 8.000001,
- "maxActiveApplicationsPerUser" : 1,
- "capacity" : 40,
- "type" : "capacitySchedulerLeafQueueInfo",
- "maxApplicationsPerUser" : 800
- }
- ],
- "state" : "RUNNING",
- "maxCapacity" : 80,
- "numApplications" : 0,
- "usedResources" : "memory: 0",
- "absoluteMaxCapacity" : 80
- },
- {
- "queueName" : "test2",
- "utilization" : 0,
- "absoluteCapacity" : 10,
- "usedCapacity" : 0,
- "capacity" : 10,
- "subQueues" : [
- {
- "numPendingApplications" : 0,
- "queueName" : "a5",
- "userLimitFactor" : 1,
- "maxApplications" : 500,
- "usedCapacity" : 0,
- "numContainers" : 0,
- "state" : "RUNNING",
- "maxCapacity" : 100,
- "numApplications" : 0,
- "usedResources" : "memory: 0",
- "absoluteMaxCapacity" : 100,
- "maxActiveApplications" : 1,
- "numActiveApplications" : 0,
- "utilization" : 0,
- "userLimit" : 100,
- "absoluteCapacity" : 5,
- "maxActiveApplicationsPerUser" : 1,
- "capacity" : 50,
- "type" : "capacitySchedulerLeafQueueInfo",
- "maxApplicationsPerUser" : 500
- },
- {
- "numPendingApplications" : 0,
- "queueName" : "a3",
- "userLimitFactor" : 1,
- "maxApplications" : 400,
- "usedCapacity" : 0,
- "numContainers" : 0,
- "state" : "RUNNING",
- "maxCapacity" : 100,
- "numApplications" : 0,
- "usedResources" : "memory: 0",
- "absoluteMaxCapacity" : 100,
- "maxActiveApplications" : 1,
- "numActiveApplications" : 0,
- "utilization" : 0,
- "userLimit" : 100,
- "absoluteCapacity" : 4.0000005,
- "maxActiveApplicationsPerUser" : 1,
- "capacity" : 40,
- "type" : "capacitySchedulerLeafQueueInfo",
- "maxApplicationsPerUser" : 400
- },
- {
- "numPendingApplications" : 0,
- "queueName" : "a4",
- "userLimitFactor" : 1,
- "maxApplications" : 100,
- "usedCapacity" : 0,
- "numContainers" : 0,
- "state" : "RUNNING",
- "maxCapacity" : 100,
- "numApplications" : 0,
- "usedResources" : "memory: 0",
- "absoluteMaxCapacity" : 100,
- "maxActiveApplications" : 1,
- "numActiveApplications" : 0,
- "utilization" : 0,
- "userLimit" : 100,
- "absoluteCapacity" : 1.0000001,
- "maxActiveApplicationsPerUser" : 1,
- "capacity" : 10,
- "type" : "capacitySchedulerLeafQueueInfo",
- "maxApplicationsPerUser" : 100
- }
- ],
- "state" : "RUNNING",
- "maxCapacity" : 15.000001,
- "numApplications" : 0,
- "usedResources" : "memory: 0",
- "absoluteMaxCapacity" : 15.000001
- }
- ],
- "usedCapacity" : 0,
- "capacity" : 100
- }
- }
- }
- +---+
- <<JSON response>>
- HTTP Request:
- -----
- Accept: application/xml
- GET http://<rm http address:port>/ws/v1/cluster/scheduler
- -----
- Response Header:
- +---+
- HTTP/1.1 200 OK
- Content-Type: application/xml
- Content-Length: 5778
- Server: Jetty(6.1.26)
- +---+
- Response Body:
- +---+
- <?xml version="1.0" encoding="UTF-8" standalone="yes"?>
- <scheduler>
- <schedulerInfo xmlns:xsi="http://www.w3.org/2001/XMLSchema-instance" xsi:type="capacityScheduler">
- <capacity>100.0</capacity>
- <usedCapacity>0.0</usedCapacity>
- <maxCapacity>100.0</maxCapacity>
- <queueName>root</queueName>
- <queues xsi:type="capacitySchedulerLeafQueueInfo">
- <capacity>70.0</capacity>
- <usedCapacity>0.0</usedCapacity>
- <maxCapacity>90.0</maxCapacity>
- <absoluteCapacity>70.0</absoluteCapacity>
- <absoluteMaxCapacity>90.0</absoluteMaxCapacity>
- <utilization>0.0</utilization>
- <numApplications>0</numApplications>
- <usedResources>memory: 0</usedResources>
- <queueName>default</queueName>
- <state>RUNNING</state>
- <numActiveApplications>0</numActiveApplications>
- <numPendingApplications>0</numPendingApplications>
- <numContainers>0</numContainers>
- <maxApplications>7000</maxApplications>
- <maxApplicationsPerUser>7000</maxApplicationsPerUser>
- <maxActiveApplications>1</maxActiveApplications>
- <maxActiveApplicationsPerUser>1</maxActiveApplicationsPerUser>
- <userLimit>100</userLimit>
- <userLimitFactor>1.0</userLimitFactor>
- </queues>
- <queues>
- <capacity>20.0</capacity>
- <usedCapacity>0.0</usedCapacity>
- <maxCapacity>80.0</maxCapacity>
- <absoluteCapacity>20.0</absoluteCapacity>
- <absoluteMaxCapacity>80.0</absoluteMaxCapacity>
- <utilization>0.0</utilization>
- <numApplications>0</numApplications>
- <usedResources>memory: 0</usedResources>
- <queueName>test</queueName>
- <state>RUNNING</state>
- <subQueues xsi:type="capacitySchedulerLeafQueueInfo">
- <capacity>60.000004</capacity>
- <usedCapacity>0.0</usedCapacity>
- <maxCapacity>80.0</maxCapacity>
- <absoluteCapacity>12.0</absoluteCapacity>
- <absoluteMaxCapacity>16.000002</absoluteMaxCapacity>
- <utilization>0.0</utilization>
- <numApplications>0</numApplications>
- <usedResources>memory: 0</usedResources>
- <queueName>a1</queueName>
- <state>RUNNING</state>
- <numActiveApplications>0</numActiveApplications>
- <numPendingApplications>0</numPendingApplications>
- <numContainers>0</numContainers>
- <maxApplications>1200</maxApplications>
- <maxApplicationsPerUser>1200</maxApplicationsPerUser>
- <maxActiveApplications>1</maxActiveApplications>
- <maxActiveApplicationsPerUser>1</maxActiveApplicationsPerUser>
- <userLimit>100</userLimit>
- <userLimitFactor>1.0</userLimitFactor>
- </subQueues>
- <subQueues xsi:type="capacitySchedulerLeafQueueInfo">
- <capacity>40.0</capacity>
- <usedCapacity>0.0</usedCapacity>
- <maxCapacity>100.0</maxCapacity>
- <absoluteCapacity>8.000001</absoluteCapacity>
- <absoluteMaxCapacity>100.0</absoluteMaxCapacity>
- <utilization>0.0</utilization>
- <numApplications>0</numApplications>
- <usedResources>memory: 0</usedResources>
- <queueName>a2</queueName>
- <state>RUNNING</state>
- <numActiveApplications>0</numActiveApplications>
- <numPendingApplications>0</numPendingApplications>
- <numContainers>0</numContainers>
- <maxApplications>800</maxApplications>
- <maxApplicationsPerUser>800</maxApplicationsPerUser>
- <maxActiveApplications>1</maxActiveApplications>
- <maxActiveApplicationsPerUser>1</maxActiveApplicationsPerUser>
- <userLimit>100</userLimit>
- <userLimitFactor>1.0</userLimitFactor>
- </subQueues>
- </queues>
- <queues>
- <capacity>10.0</capacity>
- <usedCapacity>0.0</usedCapacity>
- <maxCapacity>15.000001</maxCapacity>
- <absoluteCapacity>10.0</absoluteCapacity>
- <absoluteMaxCapacity>15.000001</absoluteMaxCapacity>
- <utilization>0.0</utilization>
- <numApplications>0</numApplications>
- <usedResources>memory: 0</usedResources>
- <queueName>test2</queueName>
- <state>RUNNING</state>
- <subQueues xsi:type="capacitySchedulerLeafQueueInfo">
- <capacity>50.0</capacity>
- <usedCapacity>0.0</usedCapacity>
- <maxCapacity>100.0</maxCapacity>
- <absoluteCapacity>5.0</absoluteCapacity>
- <absoluteMaxCapacity>100.0</absoluteMaxCapacity>
- <utilization>0.0</utilization>
- <numApplications>0</numApplications>
- <usedResources>memory: 0</usedResources>
- <queueName>A4</queueName>
- <state>RUNNING</state>
- <numActiveApplications>0</numActiveApplications>
- <numPendingApplications>0</numPendingApplications>
- <numContainers>0</numContainers>
- <maxApplications>500</maxApplications>
- <maxApplicationsPerUser>500</maxApplicationsPerUser>
- <maxActiveApplications>1</maxActiveApplications>
- <maxActiveApplicationsPerUser>1</maxActiveApplicationsPerUser>
- <userLimit>100</userLimit>
- <userLimitFactor>1.0</userLimitFactor>
- </subQueues>
- <subQueues xsi:type="capacitySchedulerLeafQueueInfo">
- <capacity>40.0</capacity>
- <usedCapacity>0.0</usedCapacity>
- <maxCapacity>100.0</maxCapacity>
- <absoluteCapacity>4.0000005</absoluteCapacity>
- <absoluteMaxCapacity>100.0</absoluteMaxCapacity>
- <utilization>0.0</utilization>
- <numApplications>0</numApplications>
- <usedResources>memory: 0</usedResources>
- <queueName>a3</queueName>
- <state>RUNNING</state>
- <numActiveApplications>0</numActiveApplications>
- <numPendingApplications>0</numPendingApplications>
- <numContainers>0</numContainers>
- <maxApplications>400</maxApplications>
- <maxApplicationsPerUser>400</maxApplicationsPerUser>
- <maxActiveApplications>1</maxActiveApplications>
- <maxActiveApplicationsPerUser>1</maxActiveApplicationsPerUser>
- <userLimit>100</userLimit>
- <userLimitFactor>1.0</userLimitFactor>
- </subQueues>
- <subQueues xsi:type="capacitySchedulerLeafQueueInfo">
- <capacity>10.0</capacity>
- <usedCapacity>0.0</usedCapacity>
- <maxCapacity>100.0</maxCapacity>
- <absoluteCapacity>1.0000001</absoluteCapacity>
- <absoluteMaxCapacity>100.0</absoluteMaxCapacity>
- <utilization>0.0</utilization>
- <numApplications>0</numApplications>
- <usedResources>memory: 0</usedResources>
- <queueName>a4</queueName>
- <state>RUNNING</state>
- <numActiveApplications>0</numActiveApplications>
- <numPendingApplications>0</numPendingApplications>
- <numContainers>0</numContainers>
- <maxApplications>100</maxApplications>
- <maxApplicationsPerUser>100</maxApplicationsPerUser>
- <maxActiveApplications>1</maxActiveApplications>
- <maxActiveApplicationsPerUser>1</maxActiveApplicationsPerUser>
- <userLimit>100</userLimit>
- <userLimitFactor>1.0</userLimitFactor>
- </subQueues>
- </queues>
- </schedulerInfo>
- </scheduler>
- +---+
- ** Fifo Scheduler API
- ** Elements of the <schedulerInfo> object
- *---------------+--------------+-------------------------------+
- || Item || Data Type || Description |
- *---------------+--------------+-------------------------------+
- | type | string | Scheduler type - fifoScheduler |
- *---------------+--------------+-------------------------------+
- | capacity | float | Queue capacity in percentage |
- *---------------+--------------+-------------------------------+
- | usedCapacity | float | Used queue capacity in percentage |
- *---------------+--------------+-------------------------------+
- | qstate | string | State of the queue - valid values are: STOPPED, RUNNING|
- *---------------+--------------+-------------------------------+
- | minQueueMemoryCapacity | int | Minimum queue memory capacity |
- *---------------+--------------+-------------------------------+
- | maxQueueMemoryCapacity | int | Maximum queue memory capacity |
- *---------------+--------------+-------------------------------+
- | numNodes | int | The total number of nodes |
- *---------------+--------------+-------------------------------+
- | usedNodeCapacity | int | The used node capacity |
- *---------------+--------------+-------------------------------+
- | availNodeCapacity | int | The available node capacity |
- *---------------+--------------+-------------------------------+
- | totalNodeCapacity | int | The total node capacity |
- *---------------+--------------+-------------------------------+
- | numContainers | int | The number of containers |
- *---------------+--------------+-------------------------------+
- *** Response Examples
- <<JSON response>>
- HTTP Request:
- ------
- GET http://<rm http address:port>/ws/v1/cluster/scheduler
- ------
- Response Header:
- +---+
- HTTP/1.1 200 OK
- Content-Type: application/json
- Transfer-Encoding: chunked
- Server: Jetty(6.1.26)
- +---+
- Response Body:
- +---+
- {
- "scheduler":
- {
- "schedulerInfo":
- {
- "type":"fifoScheduler",
- "capacity":1,
- "usedCapacity":"NaN",
- "qstate":"RUNNING",
- "minQueueMemoryCapacity":1024,
- "maxQueueMemoryCapacity":10240,
- "numNodes":0,
- "usedNodeCapacity":0,
- "availNodeCapacity":0,
- "totalNodeCapacity":0,
- "numContainers":0
- }
- }
- }
- +---+
- <<JSON response>>
- HTTP Request:
- ------
- GET http://<rm http address:port>/ws/v1/cluster/scheduler
- Accept: application/xml
- ------
- Response Header:
- +---+
- HTTP/1.1 200 OK
- Content-Type: application/xml
- Content-Length: 432
- Server: Jetty(6.1.26)
- +---+
- Response Body:
- +---+
- <?xml version="1.0" encoding="UTF-8" standalone="yes"?>
- <scheduler>
- <schedulerInfo xmlns:xsi="http://www.w3.org/2001/XMLSchema-instance" xsi:type="fifoScheduler">
- <capacity>1.0</capacity>
- <usedCapacity>NaN</usedCapacity>
- <qstate>RUNNING</qstate>
- <minQueueMemoryCapacity>1024</minQueueMemoryCapacity>
- <maxQueueMemoryCapacity>10240</maxQueueMemoryCapacity>
- <numNodes>0</numNodes>
- <usedNodeCapacity>0</usedNodeCapacity>
- <availNodeCapacity>0</availNodeCapacity>
- <totalNodeCapacity>0</totalNodeCapacity>
- <numContainers>0</numContainers>
- </schedulerInfo>
- </scheduler>
- +---+
- * Cluster Applications API
- With the Applications API, you can obtain a collection of resources, each of which represents an application. When you run a GET operation on this resource, you obtain a collection of Application Objects.
- ** URI
- ------
- * http://<rm http address:port>/ws/v1/cluster/apps
- ------
- ** HTTP Operations Supported
- ------
- * GET
- ------
- ** Query Parameters Supported
- Multiple paramters can be specified. The started and finished times have a begin and end parameter to allow you to specify ranges. For example, one could request all applications that started between 1:00am and 2:00pm on 12/19/2011 with startedTimeBegin=1324256400&startedTimeEnd=1324303200. If the Begin parameter is not specfied, it defaults to 0, and if the End parameter is not specified, it defaults to infinity.
- ------
- * state - state of the application
- * user - user name
- * queue - queue name
- * limit - total number of app objects to be returned
- * startedTimeBegin - applications with start time beginning with this time, specified in ms since epoch
- * startedTimeEnd - applications with start time ending with this time, specified in ms since epoch
- * finishedTimeBegin - applications with finish time beginning with this time, specified in ms since epoch
- * finishedTimeEnd - applications with finish time ending with this time, specified in ms since epoch
- ------
- ** Elements of the <apps> (Applications) object
- When you make a request for the list of applications, the information will be returned as a collection of app objects.
- See also {{Application API}} for syntax of the app object.
- *---------------+--------------+-------------------------------+
- || Item || Data Type || Description |
- *---------------+--------------+-------------------------------+
- | app | array of app objects(JSON)/zero or more application objects(XML) | The collection of application objects |
- *---------------+--------------+--------------------------------+
- ** Response Examples
- <<JSON response>>
- HTTP Request:
- ------
- GET http://<rm http address:port>/ws/v1/cluster/apps
- ------
- Response Header:
- +---+
- HTTP/1.1 200 OK
- Content-Type: application/json
- Transfer-Encoding: chunked
- Server: Jetty(6.1.26)
- +---+
- Response Body:
- +---+
- {
- "apps":
- {
- "app":
- [
- {
- "finishedTime" : 1326815598530,
- "amContainerLogs" : "http://host.domain.com:9999/node/containerlogs/container_1326815542473_0001_01_000001",
- "trackingUI" : "History",
- "state" : "FINISHED",
- "user" : "user1",
- "id" : "application_1326815542473_0001",
- "clusterId" : 1326815542473,
- "finalStatus" : "SUCCEEDED",
- "amHostHttpAddress" : "host.domain.com:9999",
- "progress" : 100,
- "name" : "word count",
- "startedTime" : 1326815573334,
- "elapsedTime" : 25196,
- "diagnostics" : "",
- "trackingUrl" : "http://host.domain.com:8088/proxy/application_1326815542473_0001/jobhistory/job/job_1326815542473_1_1",
- "queue" : "default"
- },
- {
- "finishedTime" : 1326815789546,
- "amContainerLogs" : "http://host.domain.com:9999/node/containerlogs/container_1326815542473_0002_01_000001",
- "trackingUI" : "History",
- "state" : "FINISHED",
- "user" : "user1",
- "id" : "application_1326815542473_0002",
- "clusterId" : 1326815542473,
- "finalStatus" : "SUCCEEDED",
- "amHostHttpAddress" : "host.domain.com:9999",
- "progress" : 100,
- "name" : "Sleep job",
- "startedTime" : 1326815641380,
- "elapsedTime" : 148166,
- "diagnostics" : "",
- "trackingUrl" : "http://host.domain.com:8088/proxy/application_1326815542473_0002/jobhistory/job/job_1326815542473_2_2",
- "queue" : "default"
- }
- ]
- }
- }
- +---+
- <<JSON response>>
- HTTP Request:
- ------
- GET http://<rm http address:port>/ws/v1/cluster/apps
- Accept: application/xml
- ------
- Response Header:
- +---+
- HTTP/1.1 200 OK
- Content-Type: application/xml
- Content-Length: 2459
- Server: Jetty(6.1.26)
- +---+
- Response Body:
- +---+
- <?xml version="1.0" encoding="UTF-8" standalone="yes"?>
- <apps>
- <app>
- <id>application_1326815542473_0001</id>
- <user>user1</user>
- <name>word count</name>
- <queue>default</queue>
- <state>FINISHED</state>
- <finalStatus>SUCCEEDED</finalStatus>
- <progress>100.0</progress>
- <trackingUI>History</trackingUI>
- <trackingUrl>http://host.domain.com:8088/proxy/application_1326815542473_0001/jobhistory/job
- /job_1326815542473_1_1</trackingUrl>
- <diagnostics/>
- <clusterId>1326815542473</clusterId>
- <startedTime>1326815573334</startedTime>
- <finishedTime>1326815598530</finishedTime>
- <elapsedTime>25196</elapsedTime>
- <amContainerLogs>http://host.domain.com:9999/node/containerlogs/container_1326815542473_0001
- _01_000001</amContainerLogs>
- <amHostHttpAddress>host.domain.com:9999</amHostHttpAddress>
- </app>
- <app>
- <id>application_1326815542473_0002</id>
- <user>user1</user>
- <name>Sleep job</name>
- <queue>default</queue>
- <state>FINISHED</state>
- <finalStatus>SUCCEEDED</finalStatus>
- <progress>100.0</progress>
- <trackingUI>History</trackingUI>
- <trackingUrl>http://host.domain.com:8088/proxy/application_1326815542473_0002/jobhistory/job/job_1326815542473_2_2</trackingUrl>
- <diagnostics/>
- <clusterId>1326815542473</clusterId>
- <startedTime>1326815641380</startedTime>
- <finishedTime>1326815789546</finishedTime>
- <elapsedTime>148166</elapsedTime>
- <amContainerLogs>http://host.domain.com:9999/node/containerlogs/container_1326815542473_0002_01_000001</amContainerLogs>
- <amHostHttpAddress>host.domain.com:9999</amHostHttpAddress>
- </app>
- </apps>
- +---+
- * Cluster {Application API}
- An application resource contains information about a particular application that was submitted to a cluster.
- ** URI
- Use the following URI to obtain an app object, from a application identified by the {appid} value.
- ------
- * http://<rm http address:port>/ws/v1/cluster/apps/{appid}
- ------
- ** HTTP Operations Supported
- ------
- * GET
- ------
- ** Query Parameters Supported
- ------
- None
- ------
- ** Elements of the <app> (Application) object
- Note that depending on security settings a user might not be able to see all the fields.
- *---------------+--------------+-------------------------------+
- || Item || Data Type || Description |
- *---------------+--------------+-------------------------------+
- | id | string | The application id |
- *---------------+--------------+--------------------------------+
- | user | string | The user who started the application |
- *---------------+--------------+--------------------------------+
- | name | string | The application name |
- *---------------+--------------+--------------------------------+
- | queue | string | The queue the application was submitted to|
- *---------------+--------------+--------------------------------+
- | state | string | The application state according to the ResourceManager - valid values are: NEW, SUBMITTED, ACCEPTED, RUNNING, FINISHED, FAILED, KILLED|
- *---------------+--------------+--------------------------------+
- | finalStatus | string | The final status of the application if finished - reported by the application itself - valid values are: UNDEFINED, SUCCEEDED, FAILED, KILLED|
- *---------------+--------------+--------------------------------+
- | progress | float | The progress of the application as a percent |
- *---------------+--------------+--------------------------------+
- | trackingUI | string | Where the tracking url is currently pointing - History (for history server) or ApplicationMaster |
- *---------------+--------------+--------------------------------+
- | trackingUrl | string | The web URL that can be used to track the application |
- *---------------+--------------+--------------------------------+
- | diagnostics | string | Detailed diagnostics information |
- *---------------+--------------+--------------------------------+
- | clusterId | long | The cluster id |
- *---------------+--------------+--------------------------------+
- | startedTime | long | The time in which application started (in ms since epoch)|
- *---------------+--------------+--------------------------------+
- | finishedTime | long | The time in which the application finished (in ms since epoch) |
- *---------------+--------------+--------------------------------+
- | elapsedTime | long | The elapsed time since the application started (in ms)|
- *---------------+--------------+--------------------------------+
- | amContainerLogs | string | The URL of the application master container logs|
- *---------------+--------------+--------------------------------+
- | amHostHttpAddress | string | The nodes http address of the application master |
- *---------------+--------------+--------------------------------+
- ** Response Examples
- <<JSON response>>
- HTTP Request:
- ------
- GET http://<rm http address:port>/ws/v1/cluster/apps/application_1326821518301_0005
- ------
- Response Header:
- +---+
- HTTP/1.1 200 OK
- Content-Type: application/json
- Transfer-Encoding: chunked
- Server: Jetty(6.1.26)
- +---+
- Response Body:
- +---+
- {
- "app" : {
- "finishedTime" : 1326824991300,
- "amContainerLogs" : "http://host.domain.com:9999/node/containerlogs/container_1326821518301_0005_01_000001",
- "trackingUI" : "History",
- "state" : "FINISHED",
- "user" : "user1",
- "id" : "application_1326821518301_0005",
- "clusterId" : 1326821518301,
- "finalStatus" : "SUCCEEDED",
- "amHostHttpAddress" : "host.domain.com:9999",
- "progress" : 100,
- "name" : "Sleep job",
- "startedTime" : 1326824544552,
- "elapsedTime" : 446748,
- "diagnostics" : "",
- "trackingUrl" : "http://host.domain.com:8088/proxy/application_1326821518301_0005/jobhistory/job/job_1326821518301_5_5",
- "queue" : "a1"
- }
- }
- +---+
- <<JSON response>>
- HTTP Request:
- ------
- GET http://<rm http address:port>/ws/v1/cluster/apps/application_1326821518301_0005
- Accept: application/xml
- ------
- Response Header:
- +---+
- HTTP/1.1 200 OK
- Content-Type: application/xml
- Content-Length: 847
- Server: Jetty(6.1.26)
- +---+
- Response Body:
- +---+
- <?xml version="1.0" encoding="UTF-8" standalone="yes"?>
- <app>
- <id>application_1326821518301_0005</id>
- <user>user1</user>
- <name>Sleep job</name>
- <queue>a1</queue>
- <state>FINISHED</state>
- <finalStatus>SUCCEEDED</finalStatus>
- <progress>100.0</progress>
- <trackingUI>History</trackingUI>
- <trackingUrl>http://host.domain.com:8088/proxy/application_1326821518301_0005/jobhistory/job/job_1326821518301_5_5</trackingUrl>
- <diagnostics/>
- <clusterId>1326821518301</clusterId>
- <startedTime>1326824544552</startedTime>
- <finishedTime>1326824991300</finishedTime>
- <elapsedTime>446748</elapsedTime>
- <amContainerLogs>http://host.domain.com:9999/node/containerlogs/container_1326821518301_0005_01_000001</amContainerLogs>
- <amHostHttpAddress>host.domain.com:9999</amHostHttpAddress>
- </app>
- +---+
- * Cluster Nodes API
- With the Nodes API, you can obtain a collection of resources, each of which represents a node. When you run a GET operation on this resource, you obtain a collection of Node Objects.
- ** URI
- ------
- * http://<rm http address:port>/ws/v1/cluster/nodes
- ------
- ** HTTP Operations Supported
- ------
- * GET
- ------
- ** Query Parameters Supported
- ------
- * state - the state of the node
- * healthy - true or false
- ------
- ** Elements of the <nodes> object
- When you make a request for the list of nodes, the information will be returned as a collection of node objects.
- See also {{Node API}} for syntax of the node object.
- *---------------+--------------+-------------------------------+
- || Item || Data Type || Description |
- *---------------+--------------+-------------------------------+
- | node | array of node objects(JSON)/zero or more node objects(XML) | A collection of node objects |
- *---------------+--------------+-------------------------------+
- ** Response Examples
- <<JSON response>>
- HTTP Request:
- ------
- GET http://<rm http address:port>/ws/v1/cluster/nodes
- ------
- Response Header:
- +---+
- HTTP/1.1 200 OK
- Content-Type: application/json
- Transfer-Encoding: chunked
- Server: Jetty(6.1.26)
- +---+
- Response Body:
- +---+
- {
- "nodes":
- {
- "node":
- [
- {
- "rack":"\/default-rack",
- "state":"NEW",
- "id":"h2:1235",
- "nodeHostName":"h2",
- "nodeHTTPAddress":"h2:2",
- "healthStatus":"Healthy",
- "lastHealthUpdate":1324056895432,
- "healthReport":"Healthy",
- "numContainers":0,
- "usedMemoryMB":0
- "availMemoryMB":8192
- },
- {
- "rack":"\/default-rack",
- "state":"NEW",
- "id":"h1:1234",
- "nodeHostName":"h1",
- "nodeHTTPAddress":"h1:2",
- "healthStatus":"Healthy",
- "lastHealthUpdate":1324056895092,
- "healthReport":"Healthy",
- "numContainers":0,
- "usedMemoryMB":0,
- "availMemoryMB":8192
- }
- ]
- }
- }
- +---+
- <<JSON response>>
- HTTP Request:
- ------
- GET http://<rm http address:port>/ws/v1/cluster/nodes
- Accept: application/xml
- ------
- Response Header:
- +---+
- HTTP/1.1 200 OK
- Content-Type: application/xml
- Content-Length: 1104
- Server: Jetty(6.1.26)
- +---+
- Response Body:
- +---+
- <?xml version="1.0" encoding="UTF-8" standalone="yes"?>
- <nodes>
- <node>
- <rack>/default-rack</rack>
- <state>RUNNING</state>
- <id>h2:1234</id>
- <nodeHostName>h2</nodeHostName>
- <nodeHTTPAddress>h2:2</nodeHTTPAddress>
- <healthStatus>Healthy</healthStatus>
- <lastHealthUpdate>1324333268447</lastHealthUpdate>
- <healthReport>Healthy</healthReport>
- <numContainers>0</numContainers>
- <usedMemoryMB>0</usedMemoryMB>
- <availMemoryMB>5120</availMemoryMB>
- </node>
- <node>
- <rack>/default-rack</rack>
- <state>RUNNING</state>
- <id>h1:1234</id>
- <nodeHostName>h1</nodeHostName>
- <nodeHTTPAddress>h1:2</nodeHTTPAddress>
- <healthStatus>Healthy</healthStatus>
- <lastHealthUpdate>1324333268447</lastHealthUpdate>
- <healthReport>Healthy</healthReport>
- <numContainers>0</numContainers>
- <usedMemoryMB>0</usedMemoryMB>
- <availMemoryMB>5120</availMemoryMB>
- </node>
- </nodes>
- +---+
- * Cluster {Node API}
- A node resource contains information about a node in the cluster.
- ** URI
- Use the following URI to obtain a Node Object, from a node identified by the {nodeid} value.
- ------
- * http://<rm http address:port>/ws/v1/cluster/nodes/{nodeid}
- ------
- ** HTTP Operations Supported
- ------
- * GET
- ------
- ** Query Parameters Supported
- ------
- None
- ------
- ** Elements of the <node> object
- *---------------+--------------+-------------------------------+
- || Item || Data Type || Description |
- *---------------+--------------+-------------------------------+
- | rack | string | The rack location of this node |
- *---------------+--------------+-------------------------------+
- | state | string | State of the node - valid values are: NEW, RUNNING, UNHEALTHY, DECOMMISSIONED, LOST, REBOOTED |
- *---------------+--------------+-------------------------------+
- | id | string | The node id |
- *---------------+--------------+-------------------------------+
- | nodeHostName | string | The host name of the node|
- *---------------+--------------+-------------------------------+
- | nodeHTTPAddress | string | The nodes HTTP address|
- *---------------+--------------+-------------------------------+
- | healthStatus | string | The health status of the node - Healthy or Unhealthy |
- *---------------+--------------+-------------------------------+
- | healthReport | string | A detailed health report |
- *---------------+--------------+-------------------------------+
- | lastHealthUpdate | long | The last time the node reported its health (in ms since epoch)|
- *---------------+--------------+-------------------------------+
- | usedMemoryMB | long | The total about of memory currently used on the node (in MB)|
- *---------------+--------------+-------------------------------+
- | availMemoryMB | long | The total amount of memory currently available on the node (in MB)|
- *---------------+--------------+-------------------------------+
- | numContainers | int | The total number of containers currently running on the node|
- *---------------+--------------+-------------------------------+
- ** Response Examples
- <<JSON response>>
- HTTP Request:
- ------
- GET http://<rm http address:port>/ws/v1/cluster/nodes/h2:1235
- ------
- Response Header:
- +---+
- HTTP/1.1 200 OK
- Content-Type: application/json
- Transfer-Encoding: chunked
- Server: Jetty(6.1.26)
- +---+
- Response Body:
- +---+
- {
- "node":
- {
- "rack":"\/default-rack",
- "state":"NEW",
- "id":"h2:1235",
- "nodeHostName":"h2",
- "nodeHTTPAddress":"h2:2",
- "healthStatus":"Healthy",
- "lastHealthUpdate":1324056895432,
- "healthReport":"Healthy",
- "numContainers":0,
- "usedMemoryMB":0,
- "availMemoryMB":5120
- }
- }
- +---+
- <<JSON response>>
- HTTP Request:
- ------
- GET http://<rm http address:port>/ws/v1/cluster/node/h2:1235
- Accept: application/xml
- ------
- Response Header:
- +---+
- HTTP/1.1 200 OK
- Content-Type: application/xml
- Content-Length: 552
- Server: Jetty(6.1.26)
- +---+
- Response Body:
- +---+
- <?xml version="1.0" encoding="UTF-8" standalone="yes"?>
- <node>
- <rack>/default-rack</rack>
- <state>NEW</state>
- <id>h2:1235</id>
- <nodeHostName>h2</nodeHostName>
- <nodeHTTPAddress>h2:2</nodeHTTPAddress>
- <healthStatus>Healthy</healthStatus>
- <lastHealthUpdate>1324333268447</lastHealthUpdate>
- <healthReport>Healthy</healthReport>
- <numContainers>0</numContainers>
- <usedMemoryMB>0</usedMemoryMB>
- <availMemoryMB>5120</availMemoryMB>
- </node>
- +---+
|