Ozone.proto 4.6 KB

123456789101112131415161718192021222324252627282930313233343536373839404142434445464748495051525354555657585960616263646566676869707172737475767778798081828384858687888990919293949596979899100101102103104105106107108109110111112113114115116117118119120121122123124125126127128129130131132133134135136137138139140141142143144145146147148149150151152153154155156
  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.ozone.protocol.proto";
  24. option java_outer_classname = "OzoneProtos";
  25. option java_generic_services = true;
  26. option java_generate_equals_and_hash = true;
  27. package hadoop.hdfs.ozone;
  28. import "hdfs.proto";
  29. // A pipeline is composed of one or more datanodes that back a container.
  30. message Pipeline {
  31. required string leaderID = 1;
  32. repeated DatanodeIDProto members = 2;
  33. required string containerName = 3;
  34. optional LifeCycleState state = 4 [default = OPEN];
  35. optional ReplicationType type = 5 [default = STAND_ALONE];
  36. optional ReplicationFactor factor = 6 [default = ONE];
  37. optional string pipelineName = 7;
  38. }
  39. message KeyValue {
  40. required string key = 1;
  41. optional string value = 2;
  42. }
  43. /**
  44. * Type of the node.
  45. */
  46. enum NodeType {
  47. KSM = 1;
  48. SCM = 2;
  49. DATANODE = 3;
  50. }
  51. // Should we rename NodeState to DatanodeState?
  52. /**
  53. * Enum that represents the Node State. This is used in calls to getNodeList
  54. * and getNodeCount.
  55. */
  56. enum NodeState {
  57. HEALTHY = 1;
  58. STALE = 2;
  59. DEAD = 3;
  60. DECOMMISSIONING = 4;
  61. DECOMMISSIONED = 5;
  62. RAFT_MEMBER = 6;
  63. FREE_NODE = 7; // Not a member in raft.
  64. UNKNOWN = 8;
  65. }
  66. enum QueryScope {
  67. CLUSTER = 1;
  68. POOL = 2;
  69. }
  70. message Node {
  71. required DatanodeIDProto nodeID = 1;
  72. repeated NodeState nodeStates = 2;
  73. }
  74. message NodePool {
  75. repeated Node nodes = 1;
  76. }
  77. /**
  78. * LifeCycleState for SCM object creation state machine:
  79. * ->Allocated: allocated on SCM but clean has not started creating it yet.
  80. * ->Creating: allocated and assigned to client to create but not ack-ed yet.
  81. * ->Open: allocated on SCM and created on datanodes and ack-ed by a client.
  82. * ->Close: container closed due to space all used or error?
  83. * ->Timeout -> container failed to create on datanodes or ack-ed by client.
  84. * ->Deleting(TBD) -> container will be deleted after timeout
  85. * 1. ALLOCATE-ed containers on SCM can't serve key/block related operation
  86. * until ACK-ed explicitly which changes the state to OPEN.
  87. * 2. Only OPEN/CLOSED containers can serve key/block related operation.
  88. * 3. ALLOCATE-ed containers that are not ACK-ed timely will be TIMEOUT and
  89. * CLEANUP asynchronously.
  90. */
  91. enum LifeCycleState {
  92. ALLOCATED = 1;
  93. CREATING = 2; // Used for container allocated/created by different client.
  94. OPEN =3; // Mostly an update to SCM via HB or client call.
  95. CLOSING = 4;
  96. CLOSED = 5; // !!State after this has not been used yet.
  97. DELETING = 6;
  98. DELETED = 7; // object is deleted.
  99. }
  100. enum LifeCycleEvent {
  101. CREATE = 1; // A request to client to create this object
  102. CREATED = 2;
  103. FINALIZE = 3;
  104. CLOSE = 4; // !!Event after this has not been used yet.
  105. UPDATE = 5;
  106. TIMEOUT = 6; // creation has timed out from SCM's View.
  107. DELETE = 7;
  108. CLEANUP = 8;
  109. }
  110. message SCMContainerInfo {
  111. // TODO : Remove the container name from pipeline.
  112. required string containerName = 1;
  113. required LifeCycleState state = 2;
  114. required Pipeline pipeline = 3;
  115. // This is not total size of container, but space allocated by SCM for
  116. // clients to write blocks
  117. required uint64 allocatedBytes = 4;
  118. required uint64 usedBytes = 5;
  119. required uint64 numberOfKeys = 6;
  120. optional int64 stateEnterTime = 7;
  121. required string owner = 8;
  122. }
  123. message GetScmInfoRequestProto {
  124. }
  125. message GetScmInfoRespsonseProto {
  126. required string clusterId = 1;
  127. required string scmId = 2;
  128. }
  129. enum ReplicationType {
  130. RATIS = 1;
  131. STAND_ALONE = 2;
  132. CHAINED = 3;
  133. }
  134. enum ReplicationFactor {
  135. ONE = 1;
  136. THREE = 3;
  137. }