StorageContainerLocationProtocol.proto 6.1 KB

123456789101112131415161718192021222324252627282930313233343536373839404142434445464748495051525354555657585960616263646566676869707172737475767778798081828384858687888990919293949596979899100101102103104105106107108109110111112113114115116117118119120121122123124125126127128129130131132133134135136137138139140141142143144145146147148149150151152153154155156157158159160161162163164165166167168169170171172173174175176177178179180181182183184185186187188189190191192193194195196197198199200201202203204205206207208209210211212213214215216217218219220221222223224225226227228229230231232233234235236237238239
  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 = "StorageContainerLocationProtocolProtos";
  25. option java_generic_services = true;
  26. option java_generate_equals_and_hash = true;
  27. package hadoop.hdds;
  28. import "hdfs.proto";
  29. import "hdds.proto";
  30. /**
  31. * Request send to SCM asking where the container should be created.
  32. */
  33. message ContainerRequestProto {
  34. // Ozone only support replciation of either 1 or 3.
  35. required ReplicationFactor replicationFactor = 2;
  36. required ReplicationType replicationType = 3;
  37. required string owner = 4;
  38. }
  39. /**
  40. * Reply from SCM indicating that the container.
  41. */
  42. message ContainerResponseProto {
  43. enum Error {
  44. success = 1;
  45. errorContainerAlreadyExists = 2;
  46. errorContainerMissing = 3;
  47. }
  48. required Error errorCode = 1;
  49. required ContainerWithPipeline containerWithPipeline = 2;
  50. optional string errorMessage = 3;
  51. }
  52. message GetContainerRequestProto {
  53. required int64 containerID = 1;
  54. }
  55. message GetContainerResponseProto {
  56. required SCMContainerInfo containerInfo = 1;
  57. }
  58. message GetContainerWithPipelineRequestProto {
  59. required int64 containerID = 1;
  60. }
  61. message GetContainerWithPipelineResponseProto {
  62. required ContainerWithPipeline containerWithPipeline = 1;
  63. }
  64. message SCMListContainerRequestProto {
  65. required uint32 count = 1;
  66. optional uint64 startContainerID = 2;
  67. }
  68. message SCMListContainerResponseProto {
  69. repeated SCMContainerInfo containers = 1;
  70. }
  71. message SCMDeleteContainerRequestProto {
  72. required int64 containerID = 1;
  73. }
  74. message SCMDeleteContainerResponseProto {
  75. // Empty response
  76. }
  77. message ObjectStageChangeRequestProto {
  78. enum Type {
  79. container = 1;
  80. pipeline = 2;
  81. }
  82. // delete/copy operation may be added later
  83. enum Op {
  84. create = 1;
  85. close = 2;
  86. }
  87. enum Stage {
  88. begin = 1;
  89. complete = 2;
  90. }
  91. required int64 id = 1;
  92. required Type type = 2;
  93. required Op op= 3;
  94. required Stage stage = 4;
  95. }
  96. message ObjectStageChangeResponseProto {
  97. // Empty response
  98. }
  99. /*
  100. NodeQueryRequest sends a request to SCM asking to send a list of nodes that
  101. match the NodeState that we are requesting.
  102. */
  103. message NodeQueryRequestProto {
  104. required NodeState state = 1;
  105. required QueryScope scope = 2;
  106. optional string poolName = 3; // if scope is pool, then pool name is needed.
  107. }
  108. message NodeQueryResponseProto {
  109. repeated Node datanodes = 1;
  110. }
  111. /**
  112. Request to create a replication pipeline.
  113. */
  114. message PipelineRequestProto {
  115. required ReplicationType replicationType = 1;
  116. required ReplicationFactor replicationFactor = 2;
  117. // if datanodes are specified then pipelines are created using those
  118. // datanodes.
  119. optional NodePool nodePool = 3;
  120. optional string pipelineID = 4;
  121. }
  122. message PipelineResponseProto {
  123. enum Error {
  124. success = 1;
  125. errorPipelineAlreadyExists = 2;
  126. }
  127. required Error errorCode = 1;
  128. optional Pipeline pipeline = 2;
  129. optional string errorMessage = 3;
  130. }
  131. message InChillModeRequestProto {
  132. }
  133. message InChillModeResponseProto {
  134. required bool inChillMode = 1;
  135. }
  136. message ForceExitChillModeRequestProto {
  137. }
  138. message ForceExitChillModeResponseProto {
  139. required bool exitedChillMode = 1;
  140. }
  141. /**
  142. * Protocol used from an HDFS node to StorageContainerManager. See the request
  143. * and response messages for details of the RPC calls.
  144. */
  145. service StorageContainerLocationProtocolService {
  146. /**
  147. * Creates a container entry in SCM.
  148. */
  149. rpc allocateContainer(ContainerRequestProto) returns (ContainerResponseProto);
  150. /**
  151. * Returns the pipeline for a given container.
  152. */
  153. rpc getContainer(GetContainerRequestProto) returns (GetContainerResponseProto);
  154. /**
  155. * Returns the pipeline for a given container.
  156. */
  157. rpc getContainerWithPipeline(GetContainerWithPipelineRequestProto) returns (GetContainerWithPipelineResponseProto);
  158. rpc listContainer(SCMListContainerRequestProto) returns (SCMListContainerResponseProto);
  159. /**
  160. * Deletes a container in SCM.
  161. */
  162. rpc deleteContainer(SCMDeleteContainerRequestProto) returns (SCMDeleteContainerResponseProto);
  163. /**
  164. * Returns a set of Nodes that meet a criteria.
  165. */
  166. rpc queryNode(NodeQueryRequestProto) returns (NodeQueryResponseProto);
  167. /**
  168. * Notify from client when begin or finish container or pipeline operations on datanodes.
  169. */
  170. rpc notifyObjectStageChange(ObjectStageChangeRequestProto) returns (ObjectStageChangeResponseProto);
  171. /*
  172. * Apis that Manage Pipelines.
  173. *
  174. * Pipelines are abstractions offered by SCM and Datanode that allows users
  175. * to create a replication pipeline.
  176. *
  177. * These following APIs allow command line programs like SCM CLI to list
  178. * and manage pipelines.
  179. */
  180. /**
  181. * Creates a replication pipeline.
  182. */
  183. rpc allocatePipeline(PipelineRequestProto)
  184. returns (PipelineResponseProto);
  185. /**
  186. * Returns information about SCM.
  187. */
  188. rpc getScmInfo(GetScmInfoRequestProto)
  189. returns (GetScmInfoRespsonseProto);
  190. /**
  191. * Checks if SCM is in ChillMode.
  192. */
  193. rpc inChillMode(InChillModeRequestProto)
  194. returns (InChillModeResponseProto);
  195. /**
  196. * Returns information about SCM.
  197. */
  198. rpc forceExitChillMode(ForceExitChillModeRequestProto)
  199. returns (ForceExitChillModeResponseProto);
  200. }