yarn_protos.proto 24 KB

123456789101112131415161718192021222324252627282930313233343536373839404142434445464748495051525354555657585960616263646566676869707172737475767778798081828384858687888990919293949596979899100101102103104105106107108109110111112113114115116117118119120121122123124125126127128129130131132133134135136137138139140141142143144145146147148149150151152153154155156157158159160161162163164165166167168169170171172173174175176177178179180181182183184185186187188189190191192193194195196197198199200201202203204205206207208209210211212213214215216217218219220221222223224225226227228229230231232233234235236237238239240241242243244245246247248249250251252253254255256257258259260261262263264265266267268269270271272273274275276277278279280281282283284285286287288289290291292293294295296297298299300301302303304305306307308309310311312313314315316317318319320321322323324325326327328329330331332333334335336337338339340341342343344345346347348349350351352353354355356357358359360361362363364365366367368369370371372373374375376377378379380381382383384385386387388389390391392393394395396397398399400401402403404405406407408409410411412413414415416417418419420421422423424425426427428429430431432433434435436437438439440441442443444445446447448449450451452453454455456457458459460461462463464465466467468469470471472473474475476477478479480481482483484485486487488489490491492493494495496497498499500501502503504505506507508509510511512513514515516517518519520521522523524525526527528529530531532533534535536537538539540541542543544545546547548549550551552553554555556557558559560561562563564565566567568569570571572573574575576577578579580581582583584585586587588589590591592593594595596597598599600601602603604605606607608609610611612613614615616617618619620621622623624625626627628629630631632633634635636637638639640641642643644645646647648649650651652653654655656657658659660661662663664665666667668669670671672673674675676677678679680681682683684685686687688689690691692693694695696697698699700701702703704705706707708709710711712713714715716717718719720721722723724725726727728729730731732733734735736737738739740741742743744745746747748749750751752753754755756757758759760761762763764765766767768769770771772773774775776777778779780781782783784785786787788789790791792793794795796797798799800801802803804805806807808809810811812813814815816817818819820821822823824825826827828829830831832833834835836837838839840841842843844
  1. /**
  2. * Licensed to the Apache Software Foundation (ASF) under one
  3. * or more contributor license agreements. See the NOTICE file
  4. * distributed with this work for additional information
  5. * regarding copyright ownership. The ASF licenses this file
  6. * to you under the Apache License, Version 2.0 (the
  7. * "License"); you may not use this file except in compliance
  8. * with the License. You may obtain a copy of the License at
  9. *
  10. * http://www.apache.org/licenses/LICENSE-2.0
  11. *
  12. * Unless required by applicable law or agreed to in writing, software
  13. * distributed under the License is distributed on an "AS IS" BASIS,
  14. * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
  15. * See the License for the specific language governing permissions and
  16. * limitations under the License.
  17. */
  18. /**
  19. * These .proto interfaces are public and stable.
  20. * Please see http://wiki.apache.org/hadoop/Compatibility
  21. * for what changes are allowed for a *stable* .proto interface.
  22. */
  23. syntax = "proto2";
  24. option java_package = "org.apache.hadoop.yarn.proto";
  25. option java_outer_classname = "YarnProtos";
  26. option java_generic_services = true;
  27. option java_generate_equals_and_hash = true;
  28. package hadoop.yarn;
  29. import "Security.proto";
  30. message SerializedExceptionProto {
  31. optional string message = 1;
  32. optional string trace = 2;
  33. optional string class_name = 3;
  34. optional SerializedExceptionProto cause = 4;
  35. }
  36. message ApplicationIdProto {
  37. optional int32 id = 1;
  38. optional int64 cluster_timestamp = 2;
  39. }
  40. message ApplicationAttemptIdProto {
  41. optional ApplicationIdProto application_id = 1;
  42. optional int32 attemptId = 2;
  43. }
  44. message ContainerIdProto {
  45. optional ApplicationIdProto app_id = 1;
  46. optional ApplicationAttemptIdProto app_attempt_id = 2;
  47. optional int64 id = 3;
  48. }
  49. enum ResourceTypesProto {
  50. COUNTABLE = 0;
  51. }
  52. message ResourceInformationProto {
  53. required string key = 1;
  54. optional int64 value = 2;
  55. optional string units = 3;
  56. optional ResourceTypesProto type = 4;
  57. repeated string tags = 5;
  58. repeated StringStringMapProto attributes = 6;
  59. }
  60. message ResourceTypeInfoProto {
  61. required string name = 1;
  62. optional string units = 2;
  63. optional ResourceTypesProto type = 3;
  64. }
  65. message ResourceProto {
  66. optional int64 memory = 1;
  67. optional int32 virtual_cores = 2;
  68. repeated ResourceInformationProto resource_value_map = 3;
  69. }
  70. message ResourceUtilizationProto {
  71. optional int32 pmem = 1;
  72. optional int32 vmem = 2;
  73. optional float cpu = 3;
  74. repeated StringFloatMapProto customResources = 4;
  75. }
  76. message ResourceOptionProto {
  77. optional ResourceProto resource = 1;
  78. optional int32 over_commit_timeout = 2;
  79. }
  80. message ResourceProfileEntry {
  81. required string name = 1;
  82. required ResourceProto resources = 2;
  83. }
  84. message ResourceProfilesProto {
  85. repeated ResourceProfileEntry resource_profiles_map = 1;
  86. }
  87. message NodeResourceMapProto {
  88. optional NodeIdProto node_id = 1;
  89. optional ResourceOptionProto resource_option = 2;
  90. }
  91. message PriorityProto {
  92. optional int32 priority = 1;
  93. }
  94. enum ContainerStateProto {
  95. C_NEW = 1;
  96. C_RUNNING = 2;
  97. C_COMPLETE = 3;
  98. }
  99. enum ContainerSubStateProto {
  100. /**
  101. * NEW, LOCALIZING, SCHEDULED,
  102. * REINITIALIZING_AWAITING_KILL, RELAUNCHING,
  103. */
  104. CSS_SCHEDULED = 1;
  105. /**
  106. * RUNNING, REINITIALIZING, PAUSING, KILLING
  107. */
  108. CSS_RUNNING = 2;
  109. /**
  110. * PAUSED, RESUMING
  111. */
  112. CSS_PAUSED = 3;
  113. /**
  114. * LOCALIZATION_FAILED, EXITED_WITH_SUCCESS,
  115. * EXITED_WITH_FAILURE,
  116. * CONTAINER_CLEANEDUP_AFTER_KILL,
  117. * CONTAINER_RESOURCES_CLEANINGUP
  118. */
  119. CSS_COMPLETING = 4;
  120. /**
  121. * DONE
  122. */
  123. CSS_DONE = 5;
  124. }
  125. message ContainerProto {
  126. optional ContainerIdProto id = 1;
  127. optional NodeIdProto nodeId = 2;
  128. optional string node_http_address = 3;
  129. optional ResourceProto resource = 4;
  130. optional PriorityProto priority = 5;
  131. optional hadoop.common.TokenProto container_token = 6;
  132. optional ExecutionTypeProto execution_type = 7 [default = GUARANTEED];
  133. optional int64 allocation_request_id = 8 [default = -1];
  134. optional int32 version = 9 [default = 0];
  135. repeated string allocation_tags = 10;
  136. optional string exposed_ports = 11;
  137. }
  138. message ContainerReportProto {
  139. optional ContainerIdProto container_id = 1;
  140. optional ResourceProto resource = 2;
  141. optional NodeIdProto node_id = 3;
  142. optional PriorityProto priority = 4;
  143. optional int64 creation_time = 5;
  144. optional int64 finish_time = 6;
  145. optional string diagnostics_info = 7 [default = "N/A"];
  146. optional string log_url = 8;
  147. optional int32 container_exit_status = 9;
  148. optional ContainerStateProto container_state = 10;
  149. optional string node_http_address = 11;
  150. optional ExecutionTypeProto executionType = 12 [default = GUARANTEED];
  151. optional string exposed_ports = 13;
  152. }
  153. enum YarnApplicationStateProto {
  154. NEW = 1;
  155. NEW_SAVING = 2;
  156. SUBMITTED = 3;
  157. ACCEPTED = 4;
  158. RUNNING = 5;
  159. FINISHED = 6;
  160. FAILED = 7;
  161. KILLED = 8;
  162. }
  163. enum YarnApplicationAttemptStateProto {
  164. APP_ATTEMPT_NEW = 1;
  165. APP_ATTEMPT_SUBMITTED = 2;
  166. APP_ATTEMPT_SCHEDULED = 3;
  167. APP_ATTEMPT_ALLOCATED_SAVING = 4;
  168. APP_ATTEMPT_ALLOCATED = 5;
  169. APP_ATTEMPT_LAUNCHED = 6;
  170. APP_ATTEMPT_FAILED = 7;
  171. APP_ATTEMPT_RUNNING = 8;
  172. APP_ATTEMPT_FINISHING = 9;
  173. APP_ATTEMPT_FINISHED = 10;
  174. APP_ATTEMPT_KILLED = 11;
  175. }
  176. enum FinalApplicationStatusProto {
  177. APP_UNDEFINED = 0;
  178. APP_SUCCEEDED = 1;
  179. APP_FAILED = 2;
  180. APP_KILLED = 3;
  181. APP_ENDED = 4;
  182. }
  183. message URLProto {
  184. optional string scheme = 1;
  185. optional string host = 2;
  186. optional int32 port = 3;
  187. optional string file = 4;
  188. optional string userInfo = 5;
  189. }
  190. enum LocalResourceVisibilityProto {
  191. PUBLIC = 1;
  192. PRIVATE = 2;
  193. APPLICATION = 3;
  194. }
  195. enum LocalResourceTypeProto {
  196. ARCHIVE = 1;
  197. FILE = 2;
  198. PATTERN = 3;
  199. }
  200. message LocalResourceProto {
  201. optional URLProto resource = 1;
  202. optional int64 size = 2;
  203. optional int64 timestamp = 3;
  204. optional LocalResourceTypeProto type = 4;
  205. optional LocalResourceVisibilityProto visibility = 5;
  206. optional string pattern = 6;
  207. optional bool should_be_uploaded_to_shared_cache = 7;
  208. }
  209. message StringLongMapProto {
  210. required string key = 1;
  211. required int64 value = 2;
  212. }
  213. message StringFloatMapProto {
  214. required string key = 1;
  215. required float value = 2;
  216. }
  217. message ApplicationResourceUsageReportProto {
  218. optional int32 num_used_containers = 1;
  219. optional int32 num_reserved_containers = 2;
  220. optional ResourceProto used_resources = 3;
  221. optional ResourceProto reserved_resources = 4;
  222. optional ResourceProto needed_resources = 5;
  223. optional int64 memory_seconds = 6;
  224. optional int64 vcore_seconds = 7;
  225. optional float queue_usage_percentage = 8;
  226. optional float cluster_usage_percentage = 9;
  227. optional int64 preempted_memory_seconds = 10;
  228. optional int64 preempted_vcore_seconds = 11;
  229. repeated StringLongMapProto application_resource_usage_map = 12;
  230. repeated StringLongMapProto application_preempted_resource_usage_map = 13;
  231. }
  232. message ApplicationReportProto {
  233. optional ApplicationIdProto applicationId = 1;
  234. optional string user = 2;
  235. optional string queue = 3;
  236. optional string name = 4;
  237. optional string host = 5;
  238. optional int32 rpc_port = 6;
  239. optional hadoop.common.TokenProto client_to_am_token = 7;
  240. optional YarnApplicationStateProto yarn_application_state = 8;
  241. optional string trackingUrl = 9;
  242. optional string diagnostics = 10 [default = "N/A"];
  243. optional int64 startTime = 11;
  244. optional int64 finishTime = 12;
  245. optional FinalApplicationStatusProto final_application_status = 13;
  246. optional ApplicationResourceUsageReportProto app_resource_Usage = 14;
  247. optional string originalTrackingUrl = 15;
  248. optional ApplicationAttemptIdProto currentApplicationAttemptId = 16;
  249. optional float progress = 17;
  250. optional string applicationType = 18;
  251. optional hadoop.common.TokenProto am_rm_token = 19;
  252. repeated string applicationTags = 20;
  253. optional LogAggregationStatusProto log_aggregation_status = 21;
  254. optional bool unmanaged_application = 22 [default = false];
  255. optional PriorityProto priority = 23;
  256. optional string appNodeLabelExpression = 24;
  257. optional string amNodeLabelExpression = 25;
  258. repeated AppTimeoutsMapProto appTimeouts = 26;
  259. optional int64 launchTime = 27;
  260. optional int64 submitTime = 28;
  261. }
  262. message AppTimeoutsMapProto {
  263. optional ApplicationTimeoutTypeProto application_timeout_type = 1;
  264. optional ApplicationTimeoutProto application_timeout = 2;
  265. }
  266. message ApplicationTimeoutProto {
  267. required ApplicationTimeoutTypeProto application_timeout_type = 1;
  268. optional string expire_time = 2;
  269. optional int64 remaining_time = 3;
  270. }
  271. enum LogAggregationStatusProto {
  272. LOG_DISABLED = 1;
  273. LOG_NOT_START = 2;
  274. LOG_RUNNING = 3;
  275. LOG_SUCCEEDED = 4;
  276. LOG_FAILED = 5;
  277. LOG_TIME_OUT = 6;
  278. LOG_RUNNING_WITH_FAILURE = 7;
  279. }
  280. message ApplicationAttemptReportProto {
  281. optional ApplicationAttemptIdProto application_attempt_id = 1;
  282. optional string host = 2;
  283. optional int32 rpc_port = 3;
  284. optional string tracking_url = 4;
  285. optional string diagnostics = 5 [default = "N/A"];
  286. optional YarnApplicationAttemptStateProto yarn_application_attempt_state = 6;
  287. optional ContainerIdProto am_container_id = 7;
  288. optional string original_tracking_url = 8;
  289. optional int64 startTime = 9;
  290. optional int64 finishTime = 10;
  291. }
  292. enum NodeStateProto {
  293. NS_NEW = 1;
  294. NS_RUNNING = 2;
  295. NS_UNHEALTHY = 3;
  296. NS_DECOMMISSIONED = 4;
  297. NS_LOST = 5;
  298. NS_REBOOTED = 6;
  299. NS_DECOMMISSIONING = 7;
  300. NS_SHUTDOWN = 8;
  301. }
  302. message NodeIdProto {
  303. optional string host = 1;
  304. optional int32 port = 2;
  305. }
  306. enum NodeUpdateTypeProto {
  307. NODE_USABLE = 0;
  308. NODE_UNUSABLE = 1;
  309. NODE_DECOMMISSIONING = 2;
  310. }
  311. message NodeReportProto {
  312. optional NodeIdProto nodeId = 1;
  313. optional string httpAddress = 2;
  314. optional string rackName = 3;
  315. optional ResourceProto used = 4;
  316. optional ResourceProto capability = 5;
  317. optional int32 numContainers = 6;
  318. optional NodeStateProto node_state = 7;
  319. optional string health_report = 8;
  320. optional int64 last_health_report_time = 9;
  321. repeated string node_labels = 10;
  322. optional ResourceUtilizationProto containers_utilization = 11;
  323. optional ResourceUtilizationProto node_utilization = 12;
  324. optional uint32 decommissioning_timeout = 13;
  325. optional NodeUpdateTypeProto node_update_type = 14;
  326. repeated NodeAttributeProto node_attributes = 15;
  327. }
  328. message NodeIdToLabelsProto {
  329. optional NodeIdProto nodeId = 1;
  330. repeated string nodeLabels = 2;
  331. }
  332. message LabelsToNodeIdsProto {
  333. optional string nodeLabels = 1;
  334. repeated NodeIdProto nodeId = 2;
  335. }
  336. message NodeLabelProto {
  337. optional string name = 1;
  338. optional bool isExclusive = 2 [default = true];
  339. }
  340. enum NodeAttributeTypeProto {
  341. STRING = 1;
  342. }
  343. message NodeAttributeKeyProto {
  344. optional string attributePrefix = 1 [default="rm.yarn.io"];
  345. required string attributeName = 2;
  346. }
  347. message NodeAttributeProto {
  348. required NodeAttributeKeyProto attributeKey = 1;
  349. optional NodeAttributeTypeProto attributeType = 2 [default = STRING];
  350. optional string attributeValue = 3 [default=""];
  351. }
  352. message NodeAttributeInfoProto {
  353. required NodeAttributeKeyProto attributeKey = 1;
  354. required NodeAttributeTypeProto attributeType = 2;
  355. }
  356. message NodeToAttributeValueProto {
  357. required string hostname = 1;
  358. required string attributeValue = 2;
  359. }
  360. message AttributeToNodesProto {
  361. required NodeAttributeKeyProto nodeAttribute = 1;
  362. repeated NodeToAttributeValueProto nodeValueMap = 2;
  363. }
  364. message NodeToAttributesProto {
  365. optional string node = 1;
  366. repeated NodeAttributeProto nodeAttributes = 2;
  367. }
  368. enum ContainerTypeProto {
  369. APPLICATION_MASTER = 1;
  370. TASK = 2;
  371. }
  372. enum ExecutionTypeProto {
  373. GUARANTEED = 1;
  374. OPPORTUNISTIC = 2;
  375. }
  376. ////////////////////////////////////////////////////////////////////////
  377. ////// From AM_RM_Protocol /////////////////////////////////////////////
  378. ////////////////////////////////////////////////////////////////////////
  379. message ResourceRequestProto {
  380. optional PriorityProto priority = 1;
  381. optional string resource_name = 2;
  382. optional ResourceProto capability = 3;
  383. optional int32 num_containers = 4;
  384. optional bool relax_locality = 5 [default = true];
  385. optional string node_label_expression = 6;
  386. optional ExecutionTypeRequestProto execution_type_request = 7;
  387. optional int64 allocation_request_id = 8 [default = -1];
  388. }
  389. message ExecutionTypeRequestProto {
  390. optional ExecutionTypeProto execution_type = 1 [default = GUARANTEED];
  391. optional bool enforce_execution_type = 2 [default = false];
  392. }
  393. message SchedulingRequestProto {
  394. optional int64 allocationRequestId = 1 [default = 0];
  395. optional PriorityProto priority = 2;
  396. optional ExecutionTypeRequestProto executionType = 3;
  397. repeated string allocationTags = 4;
  398. optional ResourceSizingProto resourceSizing = 5;
  399. optional PlacementConstraintProto placementConstraint = 6;
  400. }
  401. message ResourceSizingProto {
  402. optional int32 numAllocations = 1;
  403. optional ResourceProto resources = 2;
  404. }
  405. enum AMCommandProto {
  406. AM_RESYNC = 1;
  407. AM_SHUTDOWN = 2;
  408. }
  409. enum RejectionReasonProto {
  410. RRP_COULD_NOT_PLACE_ON_NODE = 1;
  411. RRP_COULD_NOT_SCHEDULE_ON_NODE = 2;
  412. }
  413. message RejectedSchedulingRequestProto {
  414. required RejectionReasonProto reason = 1;
  415. required SchedulingRequestProto request = 2;
  416. }
  417. message PreemptionMessageProto {
  418. optional StrictPreemptionContractProto strictContract = 1;
  419. optional PreemptionContractProto contract = 2;
  420. }
  421. message StrictPreemptionContractProto {
  422. repeated PreemptionContainerProto container = 1;
  423. }
  424. message PreemptionContractProto {
  425. repeated PreemptionResourceRequestProto resource = 1;
  426. repeated PreemptionContainerProto container = 2;
  427. }
  428. message PreemptionContainerProto {
  429. optional ContainerIdProto id = 1;
  430. }
  431. message PreemptionResourceRequestProto {
  432. optional ResourceRequestProto resource = 1;
  433. }
  434. message ResourceBlacklistRequestProto {
  435. repeated string blacklist_additions = 1;
  436. repeated string blacklist_removals = 2;
  437. }
  438. ////////////////////////////////////////////////////////////////////////
  439. ////// From client_RM_Protocol /////////////////////////////////////////
  440. ////////////////////////////////////////////////////////////////////////
  441. message ApplicationSubmissionContextProto {
  442. optional ApplicationIdProto application_id = 1;
  443. optional string application_name = 2 [default = "N/A"];
  444. optional string queue = 3 [default = "default"];
  445. optional PriorityProto priority = 4;
  446. optional ContainerLaunchContextProto am_container_spec = 5;
  447. optional bool cancel_tokens_when_complete = 6 [default = true];
  448. optional bool unmanaged_am = 7 [default = false];
  449. optional int32 maxAppAttempts = 8 [default = 0];
  450. optional ResourceProto resource = 9;
  451. optional string applicationType = 10 [default = "YARN"];
  452. optional bool keep_containers_across_application_attempts = 11 [default = false];
  453. repeated string applicationTags = 12;
  454. optional int64 attempt_failures_validity_interval = 13 [default = -1];
  455. optional LogAggregationContextProto log_aggregation_context = 14;
  456. optional ReservationIdProto reservation_id = 15;
  457. optional string node_label_expression = 16;
  458. repeated ResourceRequestProto am_container_resource_request = 17;
  459. repeated ApplicationTimeoutMapProto application_timeouts = 18;
  460. repeated StringStringMapProto application_scheduling_properties = 19;
  461. }
  462. enum ApplicationTimeoutTypeProto {
  463. APP_TIMEOUT_LIFETIME = 1;
  464. }
  465. message ApplicationTimeoutMapProto {
  466. optional ApplicationTimeoutTypeProto application_timeout_type = 1;
  467. optional int64 timeout = 2;
  468. }
  469. message ApplicationUpdateTimeoutMapProto {
  470. optional ApplicationTimeoutTypeProto application_timeout_type = 1;
  471. optional string expire_time = 2;
  472. }
  473. message LogAggregationContextProto {
  474. optional string include_pattern = 1 [default = ".*"];
  475. optional string exclude_pattern = 2 [default = ""];
  476. optional string rolled_logs_include_pattern = 3 [default = ""];
  477. optional string rolled_logs_exclude_pattern = 4 [default = ".*"];
  478. optional string log_aggregation_policy_class_name = 5;
  479. optional string log_aggregation_policy_parameters = 6;
  480. }
  481. enum ApplicationAccessTypeProto {
  482. APPACCESS_VIEW_APP = 1;
  483. APPACCESS_MODIFY_APP = 2;
  484. }
  485. message ApplicationACLMapProto {
  486. optional ApplicationAccessTypeProto accessType = 1;
  487. optional string acl = 2 [default = " "];
  488. }
  489. message YarnClusterMetricsProto {
  490. optional int32 num_node_managers = 1;
  491. optional int32 num_decommissioned_nms = 2;
  492. optional int32 num_active_nms = 3;
  493. optional int32 num_lost_nms = 4;
  494. optional int32 num_unhealthy_nms = 5;
  495. optional int32 num_rebooted_nms = 6;
  496. }
  497. enum QueueStateProto {
  498. Q_STOPPED = 1;
  499. Q_RUNNING = 2;
  500. Q_DRAINING = 3;
  501. }
  502. message QueueStatisticsProto {
  503. optional int64 numAppsSubmitted = 1;
  504. optional int64 numAppsRunning = 2;
  505. optional int64 numAppsPending = 3;
  506. optional int64 numAppsCompleted = 4;
  507. optional int64 numAppsKilled = 5;
  508. optional int64 numAppsFailed = 6;
  509. optional int64 numActiveUsers = 7;
  510. optional int64 availableMemoryMB = 8;
  511. optional int64 allocatedMemoryMB = 9;
  512. optional int64 pendingMemoryMB = 10;
  513. optional int64 reservedMemoryMB = 11;
  514. optional int64 availableVCores = 12;
  515. optional int64 allocatedVCores = 13;
  516. optional int64 pendingVCores = 14;
  517. optional int64 reservedVCores = 15;
  518. optional int64 allocatedContainers = 16;
  519. optional int64 pendingContainers = 17;
  520. optional int64 reservedContainers = 18;
  521. }
  522. message QueueInfoProto {
  523. optional string queueName = 1;
  524. optional float capacity = 2;
  525. optional float maximumCapacity = 3;
  526. optional float currentCapacity = 4;
  527. optional QueueStateProto state = 5;
  528. repeated QueueInfoProto childQueues = 6;
  529. repeated ApplicationReportProto applications = 7;
  530. repeated string accessibleNodeLabels = 8;
  531. optional string defaultNodeLabelExpression = 9;
  532. optional QueueStatisticsProto queueStatistics = 10;
  533. optional bool preemptionDisabled = 11;
  534. repeated QueueConfigurationsMapProto queueConfigurationsMap = 12;
  535. optional bool intraQueuePreemptionDisabled = 13;
  536. }
  537. message QueueConfigurationsProto {
  538. optional float capacity = 1;
  539. optional float absoluteCapacity = 2;
  540. optional float maxCapacity = 3;
  541. optional float absoluteMaxCapacity = 4;
  542. optional float maxAMPercentage = 5;
  543. optional ResourceProto effectiveMinCapacity = 6;
  544. optional ResourceProto effectiveMaxCapacity = 7;
  545. optional ResourceProto configuredMinCapacity = 8;
  546. optional ResourceProto configuredMaxCapacity = 9;
  547. }
  548. message QueueConfigurationsMapProto {
  549. required string partitionName = 1;
  550. optional QueueConfigurationsProto queueConfigurations = 2;
  551. }
  552. enum QueueACLProto {
  553. QACL_SUBMIT_APPLICATIONS = 1;
  554. QACL_ADMINISTER_QUEUE = 2;
  555. }
  556. message QueueUserACLInfoProto {
  557. optional string queueName = 1;
  558. repeated QueueACLProto userAcls = 2;
  559. }
  560. enum SignalContainerCommandProto {
  561. OUTPUT_THREAD_DUMP = 1;
  562. GRACEFUL_SHUTDOWN = 2;
  563. FORCEFUL_SHUTDOWN = 3;
  564. }
  565. ////////////////////////////////////////////////////////////////////////
  566. ////// Placement constraints ///////////////////////////////////////////
  567. ////////////////////////////////////////////////////////////////////////
  568. message PlacementConstraintProto {
  569. optional SimplePlacementConstraintProto simpleConstraint = 1;
  570. optional CompositePlacementConstraintProto compositeConstraint = 2;
  571. }
  572. enum NodeAttributeOpCodeProto {
  573. NO_OP = 1;
  574. EQ = 2;
  575. NE = 3;
  576. }
  577. message SimplePlacementConstraintProto {
  578. required string scope = 1;
  579. repeated PlacementConstraintTargetProto targetExpressions = 2;
  580. optional int32 minCardinality = 3;
  581. optional int32 maxCardinality = 4;
  582. optional NodeAttributeOpCodeProto attributeOpCode = 5;
  583. }
  584. message PlacementConstraintTargetProto {
  585. enum TargetType {
  586. NODE_ATTRIBUTE = 1;
  587. ALLOCATION_TAG = 2;
  588. SELF = 3;
  589. }
  590. required TargetType targetType = 1;
  591. optional string targetKey = 2;
  592. repeated string targetValues = 3;
  593. }
  594. message TimedPlacementConstraintProto {
  595. enum DelayUnit {
  596. MILLISECONDS = 1;
  597. OPPORTUNITIES = 2;
  598. }
  599. required PlacementConstraintProto placementConstraint = 1;
  600. required int64 schedulingDelay = 2;
  601. optional DelayUnit delayUnit = 3 [ default = MILLISECONDS ];
  602. }
  603. message CompositePlacementConstraintProto {
  604. enum CompositeType {
  605. // All children constraints have to be satisfied.
  606. AND = 1;
  607. // One of the children constraints has to be satisfied.
  608. OR = 2;
  609. // Attempt to satisfy the first child constraint for delays[0] units (e.g.,
  610. // millisec or heartbeats). If this fails, try to satisfy the second child
  611. // constraint for delays[1] units and so on.
  612. DELAYED_OR = 3;
  613. }
  614. required CompositeType compositeType = 1;
  615. repeated PlacementConstraintProto childConstraints = 2;
  616. repeated TimedPlacementConstraintProto timedChildConstraints = 3;
  617. }
  618. // This associates a set of allocation tags to a Placement Constraint.
  619. message PlacementConstraintMapEntryProto {
  620. repeated string allocation_tags = 1;
  621. optional PlacementConstraintProto placement_constraint = 2;
  622. }
  623. ////////////////////////////////////////////////////////////////////////
  624. ////// From reservation_protocol /////////////////////////////////////
  625. ////////////////////////////////////////////////////////////////////////
  626. message ReservationIdProto {
  627. optional int64 id = 1;
  628. optional int64 cluster_timestamp = 2;
  629. }
  630. message ReservationRequestProto {
  631. optional ResourceProto capability = 1;
  632. optional int32 num_containers = 2 [default = 1];
  633. optional int32 concurrency = 3 [default = 1];
  634. optional int64 duration = 4 [default = -1];
  635. }
  636. message ReservationRequestsProto {
  637. repeated ReservationRequestProto reservation_resources = 1;
  638. optional ReservationRequestInterpreterProto interpreter = 2 [default = R_ALL];
  639. }
  640. message ReservationDefinitionProto {
  641. optional ReservationRequestsProto reservation_requests = 1;
  642. optional int64 arrival = 2;
  643. optional int64 deadline = 3;
  644. optional string reservation_name = 4;
  645. optional string recurrence_expression = 5 [default = "0"];
  646. optional PriorityProto priority = 6;
  647. }
  648. message ResourceAllocationRequestProto {
  649. optional int64 start_time = 1;
  650. optional int64 end_time = 2;
  651. optional ResourceProto resource = 3;
  652. }
  653. message ReservationAllocationStateProto {
  654. optional ReservationDefinitionProto reservation_definition = 1;
  655. repeated ResourceAllocationRequestProto allocation_requests = 2;
  656. optional int64 start_time = 3;
  657. optional int64 end_time = 4;
  658. optional string user = 5;
  659. optional bool contains_gangs = 6;
  660. optional int64 acceptance_time = 7;
  661. optional ReservationIdProto reservation_id = 8;
  662. }
  663. enum ReservationRequestInterpreterProto {
  664. R_ANY = 0;
  665. R_ALL = 1;
  666. R_ORDER = 2;
  667. R_ORDER_NO_GAP = 3;
  668. }
  669. ////////////////////////////////////////////////////////////////////////
  670. ////// From container_manager //////////////////////////////////////////
  671. ////////////////////////////////////////////////////////////////////////
  672. message ContainerLaunchContextProto {
  673. repeated StringLocalResourceMapProto localResources = 1;
  674. optional bytes tokens = 2;
  675. repeated StringBytesMapProto service_data = 3;
  676. repeated StringStringMapProto environment = 4;
  677. repeated string command = 5;
  678. repeated ApplicationACLMapProto application_ACLs = 6;
  679. optional ContainerRetryContextProto container_retry_context = 7;
  680. optional bytes tokens_conf = 8;
  681. }
  682. message ContainerStatusProto {
  683. optional ContainerIdProto container_id = 1;
  684. optional ContainerStateProto state = 2;
  685. optional string diagnostics = 3 [default = "N/A"];
  686. optional int32 exit_status = 4 [default = -1000];
  687. optional ResourceProto capability = 5;
  688. optional ExecutionTypeProto executionType = 6 [default = GUARANTEED];
  689. repeated StringStringMapProto container_attributes = 7;
  690. optional ContainerSubStateProto container_sub_state = 8;
  691. }
  692. enum ContainerExitStatusProto {
  693. SUCCESS = 0;
  694. INVALID = -1000;
  695. ABORTED = -100;
  696. DISKS_FAILED = -101;
  697. }
  698. message ContainerRetryContextProto {
  699. optional ContainerRetryPolicyProto retry_policy = 1 [default = NEVER_RETRY];
  700. repeated int32 error_codes = 2;
  701. optional int32 max_retries = 3 [default = 0];
  702. optional int32 retry_interval = 4 [default = 0];
  703. optional int64 failures_validity_interval = 5 [default = -1];
  704. }
  705. enum ContainerRetryPolicyProto {
  706. NEVER_RETRY = 0;
  707. RETRY_ON_ALL_ERRORS = 1;
  708. RETRY_ON_SPECIFIC_ERROR_CODES = 2;
  709. }
  710. ////////////////////////////////////////////////////////////////////////
  711. ////// From common//////////////////////////////////////////////////////
  712. ////////////////////////////////////////////////////////////////////////
  713. message StringLocalResourceMapProto {
  714. optional string key = 1;
  715. optional LocalResourceProto value = 2;
  716. }
  717. message StringStringMapProto {
  718. optional string key = 1;
  719. optional string value = 2;
  720. }
  721. message StringBytesMapProto {
  722. optional string key = 1;
  723. optional bytes value = 2;
  724. }
  725. message CollectorInfoProto {
  726. optional string collector_addr = 1;
  727. optional hadoop.common.TokenProto collector_token = 2;
  728. }