yarn_protos.proto 24 KB

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