hdds.proto 6.6 KB

123456789101112131415161718192021222324252627282930313233343536373839404142434445464748495051525354555657585960616263646566676869707172737475767778798081828384858687888990919293949596979899100101102103104105106107108109110111112113114115116117118119120121122123124125126127128129130131132133134135136137138139140141142143144145146147148149150151152153154155156157158159160161162163164165166167168169170171172173174175176177178179180181182183184185186187188189190191192193194195196197198199200201202203204205206207208209210211212213214215216217218219220221222223224225226227228229230231232233234235236237238239240241242243244245246247248249
  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 private and unstable.
  20. * Please see http://wiki.apache.org/hadoop/Compatibility
  21. * for what changes are allowed for a *unstable* .proto interface.
  22. */
  23. option java_package = "org.apache.hadoop.hdds.protocol.proto";
  24. option java_outer_classname = "HddsProtos";
  25. option java_generic_services = true;
  26. option java_generate_equals_and_hash = true;
  27. package hadoop.hdds;
  28. message DatanodeDetailsProto {
  29. required string uuid = 1; // UUID assigned to the Datanode.
  30. required string ipAddress = 2; // IP address
  31. required string hostName = 3; // hostname
  32. repeated Port ports = 4;
  33. optional string certSerialId = 5; // Certificate serial id.
  34. // network name, can be Ip address or host name, depends
  35. optional string networkName = 6;
  36. optional string networkLocation = 7; // Network topology location
  37. }
  38. /**
  39. Proto message encapsulating information required to uniquely identify a
  40. OzoneManager.
  41. */
  42. message OzoneManagerDetailsProto {
  43. required string uuid = 1; // UUID assigned to the OzoneManager.
  44. required string ipAddress = 2; // IP address of OM.
  45. required string hostName = 3; // Hostname of OM.
  46. repeated Port ports = 4;
  47. }
  48. message Port {
  49. required string name = 1;
  50. required uint32 value = 2;
  51. }
  52. message PipelineID {
  53. required string id = 1;
  54. }
  55. enum PipelineState {
  56. PIPELINE_ALLOCATED = 1;
  57. PIPELINE_OPEN = 2;
  58. PIPELINE_DORMANT = 3;
  59. PIPELINE_CLOSED = 4;
  60. }
  61. message Pipeline {
  62. required string leaderID = 1;
  63. repeated DatanodeDetailsProto members = 2;
  64. // TODO: remove the state and leaderID from this class
  65. optional PipelineState state = 3 [default = PIPELINE_ALLOCATED];
  66. optional ReplicationType type = 4 [default = STAND_ALONE];
  67. optional ReplicationFactor factor = 5 [default = ONE];
  68. required PipelineID id = 6;
  69. repeated uint32 memberOrders = 7;
  70. }
  71. message KeyValue {
  72. required string key = 1;
  73. optional string value = 2;
  74. }
  75. /**
  76. * Type of the node.
  77. */
  78. enum NodeType {
  79. OM = 1; // Ozone Manager
  80. SCM = 2; // Storage Container Manager
  81. DATANODE = 3; // DataNode
  82. }
  83. // Should we rename NodeState to DatanodeState?
  84. /**
  85. * Enum that represents the Node State. This is used in calls to getNodeList
  86. * and getNodeCount.
  87. */
  88. enum NodeState {
  89. HEALTHY = 1;
  90. STALE = 2;
  91. DEAD = 3;
  92. DECOMMISSIONING = 4;
  93. DECOMMISSIONED = 5;
  94. }
  95. enum QueryScope {
  96. CLUSTER = 1;
  97. POOL = 2;
  98. }
  99. message Node {
  100. required DatanodeDetailsProto nodeID = 1;
  101. repeated NodeState nodeStates = 2;
  102. }
  103. message NodePool {
  104. repeated Node nodes = 1;
  105. }
  106. /**
  107. * LifeCycleState for SCM object creation state machine:
  108. * ->Allocated: allocated on SCM but clean has not started creating it yet.
  109. * ->Creating: allocated and assigned to client to create but not ack-ed yet.
  110. * ->Open: allocated on SCM and created on datanodes and ack-ed by a client.
  111. * ->Close: container closed due to space all used or error?
  112. * ->Timeout -> container failed to create on datanodes or ack-ed by client.
  113. * ->Deleting(TBD) -> container will be deleted after timeout
  114. * 1. ALLOCATE-ed containers on SCM can't serve key/block related operation
  115. * until ACK-ed explicitly which changes the state to OPEN.
  116. * 2. Only OPEN/CLOSED containers can serve key/block related operation.
  117. * 3. ALLOCATE-ed containers that are not ACK-ed timely will be TIMEOUT and
  118. * CLEANUP asynchronously.
  119. */
  120. enum LifeCycleState {
  121. OPEN = 1;
  122. CLOSING = 2;
  123. QUASI_CLOSED = 3;
  124. CLOSED = 4;
  125. DELETING = 5;
  126. DELETED = 6; // object is deleted.
  127. }
  128. enum LifeCycleEvent {
  129. FINALIZE = 1;
  130. QUASI_CLOSE = 2;
  131. CLOSE = 3; // !!Event after this has not been used yet.
  132. FORCE_CLOSE = 4;
  133. DELETE = 5;
  134. CLEANUP = 6;
  135. }
  136. message ContainerInfoProto {
  137. required int64 containerID = 1;
  138. required LifeCycleState state = 2;
  139. optional PipelineID pipelineID = 3;
  140. required uint64 usedBytes = 4;
  141. required uint64 numberOfKeys = 5;
  142. optional int64 stateEnterTime = 6;
  143. required string owner = 7;
  144. optional int64 deleteTransactionId = 8;
  145. optional int64 sequenceId = 9;
  146. required ReplicationFactor replicationFactor = 10;
  147. required ReplicationType replicationType = 11;
  148. }
  149. message ContainerWithPipeline {
  150. required ContainerInfoProto containerInfo = 1;
  151. required Pipeline pipeline = 2;
  152. }
  153. message GetScmInfoRequestProto {
  154. optional string traceID = 1;
  155. }
  156. message GetScmInfoResponseProto {
  157. required string clusterId = 1;
  158. required string scmId = 2;
  159. }
  160. enum ReplicationType {
  161. RATIS = 1;
  162. STAND_ALONE = 2;
  163. CHAINED = 3;
  164. }
  165. enum ReplicationFactor {
  166. ONE = 1;
  167. THREE = 3;
  168. }
  169. enum ScmOps {
  170. allocateBlock = 1;
  171. keyBlocksInfoList = 2;
  172. getScmInfo = 3;
  173. deleteBlock = 4;
  174. createReplicationPipeline = 5;
  175. allocateContainer = 6;
  176. getContainer = 7;
  177. getContainerWithPipeline = 8;
  178. listContainer = 9;
  179. deleteContainer = 10;
  180. queryNode = 11;
  181. }
  182. message ExcludeListProto {
  183. repeated string datanodes = 1;
  184. repeated int64 containerIds = 2;
  185. repeated PipelineID pipelineIds = 3;
  186. }
  187. /**
  188. * Block ID that uniquely identify a block by SCM.
  189. */
  190. message ContainerBlockID {
  191. required int64 containerID = 1;
  192. required int64 localID = 2;
  193. }
  194. /**
  195. * Information for the Hdds block token.
  196. * When adding further fields, make sure they are optional as they would
  197. * otherwise not be backwards compatible.
  198. */
  199. message BlockTokenSecretProto {
  200. /**
  201. * File access permissions mode.
  202. */
  203. enum AccessModeProto {
  204. READ = 1;
  205. WRITE = 2;
  206. COPY = 3;
  207. DELETE = 4;
  208. }
  209. required string ownerId = 1;
  210. required string blockId = 2;
  211. required uint64 expiryDate = 3;
  212. required string omCertSerialId = 4;
  213. repeated AccessModeProto modes = 5;
  214. required uint64 maxLength = 6;
  215. }
  216. message BlockID {
  217. required ContainerBlockID containerBlockID = 1;
  218. optional uint64 blockCommitSequenceId = 2 [default = 0];
  219. }