mr_protos.proto 4.8 KB

123456789101112131415161718192021222324252627282930313233343536373839404142434445464748495051525354555657585960616263646566676869707172737475767778798081828384858687888990919293949596979899100101102103104105106107108109110111112113114115116117118119120121122123124125126127128129130131132133134135136137138139140141142143144145146147148149150151152153154155156157158159160161162163164165166167168169170171172173174175176177178179180181182183184
  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. option java_package = "org.apache.hadoop.mapreduce.v2.proto";
  19. option java_outer_classname = "MRProtos";
  20. option java_generic_services = true;
  21. option java_generate_equals_and_hash = true;
  22. package hadoop.mapreduce;
  23. import "yarn_protos.proto";
  24. enum TaskTypeProto {
  25. MAP = 1;
  26. REDUCE = 2;
  27. }
  28. message JobIdProto {
  29. optional hadoop.yarn.ApplicationIdProto app_id = 1;
  30. optional int32 id = 2;
  31. }
  32. message TaskIdProto {
  33. optional JobIdProto job_id = 1;
  34. optional TaskTypeProto task_type = 2;
  35. optional int32 id = 3;
  36. }
  37. message TaskAttemptIdProto {
  38. optional TaskIdProto task_id = 1;
  39. optional int32 id = 2;
  40. }
  41. enum TaskStateProto {
  42. TS_NEW = 1;
  43. TS_SCHEDULED = 2;
  44. TS_RUNNING = 3;
  45. TS_SUCCEEDED = 4;
  46. TS_FAILED = 5;
  47. TS_KILLED = 6;
  48. }
  49. enum PhaseProto {
  50. P_STARTING = 1;
  51. P_MAP = 2;
  52. P_SHUFFLE = 3;
  53. P_SORT = 4;
  54. P_REDUCE = 5;
  55. P_CLEANUP = 6;
  56. }
  57. message CounterProto {
  58. optional string name = 1;
  59. optional string display_name = 2;
  60. optional int64 value = 3;
  61. }
  62. message CounterGroupProto {
  63. optional string name = 1;
  64. optional string display_name = 2;
  65. repeated StringCounterMapProto counters = 3;
  66. }
  67. message CountersProto {
  68. repeated StringCounterGroupMapProto counter_groups = 1;
  69. }
  70. message TaskReportProto {
  71. optional TaskIdProto task_id = 1;
  72. optional TaskStateProto task_state = 2;
  73. optional float progress = 3;
  74. optional int64 start_time = 4;
  75. optional int64 finish_time = 5;
  76. optional CountersProto counters = 6;
  77. repeated TaskAttemptIdProto running_attempts = 7;
  78. optional TaskAttemptIdProto successful_attempt = 8;
  79. repeated string diagnostics = 9;
  80. }
  81. enum TaskAttemptStateProto {
  82. TA_NEW = 1;
  83. TA_STARTING = 2;
  84. TA_RUNNING = 3;
  85. TA_COMMIT_PENDING = 4;
  86. TA_SUCCEEDED = 5;
  87. TA_FAILED = 6;
  88. TA_KILLED = 7;
  89. }
  90. message TaskAttemptReportProto {
  91. optional TaskAttemptIdProto task_attempt_id = 1;
  92. optional TaskAttemptStateProto task_attempt_state = 2;
  93. optional float progress = 3;
  94. optional int64 start_time = 4;
  95. optional int64 finish_time = 5;
  96. optional CountersProto counters = 6;
  97. optional string diagnostic_info = 7;
  98. optional string state_string = 8;
  99. optional PhaseProto phase = 9;
  100. optional int64 shuffle_finish_time = 10;
  101. optional int64 sort_finish_time=11;
  102. optional string node_manager_host = 12;
  103. optional int32 node_manager_port = 13;
  104. optional int32 node_manager_http_port = 14;
  105. optional hadoop.yarn.ContainerIdProto container_id = 15;
  106. }
  107. enum JobStateProto {
  108. J_NEW = 1;
  109. J_INITED = 2;
  110. J_RUNNING = 3;
  111. J_SUCCEEDED = 4;
  112. J_FAILED = 5;
  113. J_KILLED = 6;
  114. J_ERROR = 7;
  115. }
  116. message JobReportProto {
  117. optional JobIdProto job_id = 1;
  118. optional JobStateProto job_state = 2;
  119. optional float map_progress = 3;
  120. optional float reduce_progress = 4;
  121. optional float cleanup_progress = 5;
  122. optional float setup_progress = 6;
  123. optional int64 start_time = 7;
  124. optional int64 finish_time = 8;
  125. optional string user = 9;
  126. optional string jobName = 10;
  127. optional string trackingUrl = 11;
  128. optional string diagnostics = 12;
  129. optional string jobFile = 13;
  130. repeated AMInfoProto am_infos = 14;
  131. optional int64 submit_time = 15;
  132. optional bool is_uber = 16 [default = false];
  133. }
  134. message AMInfoProto {
  135. optional hadoop.yarn.ApplicationAttemptIdProto application_attempt_id = 1;
  136. optional int64 start_time = 2;
  137. optional hadoop.yarn.ContainerIdProto container_id = 3;
  138. optional string node_manager_host = 4;
  139. optional int32 node_manager_port = 5;
  140. optional int32 node_manager_http_port = 6;
  141. }
  142. enum TaskAttemptCompletionEventStatusProto {
  143. TACE_FAILED = 1;
  144. TACE_KILLED = 2;
  145. TACE_SUCCEEDED = 3;
  146. TACE_OBSOLETE = 4;
  147. TACE_TIPFAILED = 5;
  148. }
  149. message TaskAttemptCompletionEventProto {
  150. optional TaskAttemptIdProto attempt_id = 1;
  151. optional TaskAttemptCompletionEventStatusProto status = 2;
  152. optional string map_output_server_address = 3;
  153. optional int32 attempt_run_time = 4;
  154. optional int32 event_id = 5;
  155. }
  156. message StringCounterMapProto {
  157. optional string key = 1;
  158. optional CounterProto value = 2;
  159. }
  160. message StringCounterGroupMapProto {
  161. optional string key = 1;
  162. optional CounterGroupProto value = 2;
  163. }