StorageContainerDatanodeProtocol.proto 10 KB

123456789101112131415161718192021222324252627282930313233343536373839404142434445464748495051525354555657585960616263646566676869707172737475767778798081828384858687888990919293949596979899100101102103104105106107108109110111112113114115116117118119120121122123124125126127128129130131132133134135136137138139140141142143144145146147148149150151152153154155156157158159160161162163164165166167168169170171172173174175176177178179180181182183184185186187188189190191192193194195196197198199200201202203204205206207208209210211212213214215216217218219220221222223224225226227228229230231232233234235236237238239240241242243244245246247248249250251252253254255256257258259260261262263264265266267268269270271272273274275276277278279280281282283284285286287288289290291292293294295296297298299300301302303304305306307308309310311312313314315316317318319320321322323324325326327328329330331332333334335336337338339340341342343344345346347348349350351352353
  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 = "StorageContainerDatanodeProtocolProtos";
  25. option java_generic_services = true;
  26. option java_generate_equals_and_hash = true;
  27. package hadoop.hdds;
  28. import "hdds.proto";
  29. /**
  30. * This message is send by data node to indicate that it is alive or it is
  31. * registering with the node manager.
  32. */
  33. message SCMHeartbeatRequestProto {
  34. required DatanodeDetailsProto datanodeDetails = 1;
  35. optional SCMNodeReport nodeReport = 2;
  36. optional ReportState containerReportState = 3;
  37. }
  38. enum DatanodeContainerState {
  39. closed = 0;
  40. open = 1;
  41. }
  42. /**
  43. NodeState contains messages from datanode to SCM saying that it has
  44. some information that SCM might be interested in.*/
  45. message ReportState {
  46. enum states {
  47. noContainerReports = 0;
  48. completeContinerReport = 1;
  49. deltaContainerReport = 2;
  50. }
  51. required states state = 1;
  52. required int64 count = 2 [default = 0];
  53. }
  54. /**
  55. This message is used to persist the information about a container in the
  56. SCM database, This information allows SCM to startup faster and avoid having
  57. all container info in memory all the time.
  58. */
  59. message ContainerPersistanceProto {
  60. required DatanodeContainerState state = 1;
  61. required hadoop.hdds.Pipeline pipeline = 2;
  62. required ContainerInfo info = 3;
  63. }
  64. /**
  65. This message is used to do a quick look up of which containers are effected
  66. if a node goes down
  67. */
  68. message NodeContianerMapping {
  69. repeated string contianerName = 1;
  70. }
  71. /**
  72. A container report contains the following information.
  73. */
  74. message ContainerInfo {
  75. required string containerName = 1;
  76. optional string finalhash = 2;
  77. optional int64 size = 3;
  78. optional int64 used = 4;
  79. optional int64 keyCount = 5;
  80. // TODO: move the io count to separate message
  81. optional int64 readCount = 6;
  82. optional int64 writeCount = 7;
  83. optional int64 readBytes = 8;
  84. optional int64 writeBytes = 9;
  85. required int64 containerID = 10;
  86. optional hadoop.hdds.LifeCycleState state = 11;
  87. }
  88. // The deleted blocks which are stored in deletedBlock.db of scm.
  89. message DeletedBlocksTransaction {
  90. required int64 txID = 1;
  91. required string containerName = 2;
  92. repeated string blockID = 3;
  93. // the retry time of sending deleting command to datanode.
  94. required int32 count = 4;
  95. }
  96. /**
  97. A set of container reports, max count is generally set to
  98. 8192 since that keeps the size of the reports under 1 MB.
  99. */
  100. message ContainerReportsRequestProto {
  101. enum reportType {
  102. fullReport = 0;
  103. deltaReport = 1;
  104. }
  105. required DatanodeDetailsProto datanodeDetails = 1;
  106. repeated ContainerInfo reports = 2;
  107. required reportType type = 3;
  108. }
  109. message ContainerReportsResponseProto {
  110. }
  111. /**
  112. * This message is send along with the heart beat to report datanode
  113. * storage utilization by SCM.
  114. */
  115. message SCMNodeReport {
  116. repeated SCMStorageReport storageReport = 1;
  117. }
  118. message SCMStorageReport {
  119. required string storageUuid = 1;
  120. optional uint64 capacity = 2 [default = 0];
  121. optional uint64 scmUsed = 3 [default = 0];
  122. optional uint64 remaining = 4 [default = 0];
  123. //optional hadoop.hdfs.StorageTypeProto storageType = 5 [default = DISK];
  124. }
  125. message SCMRegisterRequestProto {
  126. required DatanodeDetailsProto datanodeDetails = 1;
  127. optional SCMNodeAddressList addressList = 2;
  128. }
  129. /**
  130. * Request for version info of the software stack on the server.
  131. */
  132. message SCMVersionRequestProto {
  133. }
  134. /**
  135. * Generic response that is send to a version request. This allows keys to be
  136. * added on the fly and protocol to remain stable.
  137. */
  138. message SCMVersionResponseProto {
  139. required uint32 softwareVersion = 1;
  140. repeated hadoop.hdds.KeyValue keys = 2;
  141. }
  142. message SCMNodeAddressList {
  143. repeated string addressList = 1;
  144. }
  145. /**
  146. * Datanode ID returned by the SCM. This is similar to name node
  147. * registeration of a datanode.
  148. */
  149. message SCMRegisteredCmdResponseProto {
  150. enum ErrorCode {
  151. success = 1;
  152. errorNodeNotPermitted = 2;
  153. }
  154. required ErrorCode errorCode = 2;
  155. required string datanodeUUID = 3;
  156. required string clusterID = 4;
  157. optional SCMNodeAddressList addressList = 5;
  158. optional string hostname = 6;
  159. optional string ipAddress = 7;
  160. }
  161. /**
  162. * SCM informs a datanode to register itself again.
  163. * With recieving this command, datanode will transit to REGISTER state.
  164. */
  165. message SCMReregisterCmdResponseProto {}
  166. /**
  167. This command tells the data node to send in the container report when possible
  168. */
  169. message SendContainerReportProto {
  170. }
  171. /**
  172. This command asks the datanode to close a specific container.
  173. */
  174. message SCMCloseContainerCmdResponseProto {
  175. required string containerName = 1;
  176. }
  177. /**
  178. Type of commands supported by SCM to datanode protocol.
  179. */
  180. enum SCMCmdType {
  181. versionCommand = 2;
  182. registeredCommand = 3;
  183. sendContainerReport = 4;
  184. reregisterCommand = 5;
  185. deleteBlocksCommand = 6;
  186. closeContainerCommand = 7;
  187. }
  188. /*
  189. * These are commands returned by SCM for to the datanode to execute.
  190. */
  191. message SCMCommandResponseProto {
  192. required SCMCmdType cmdType = 2; // Type of the command
  193. optional SCMRegisteredCmdResponseProto registeredProto = 3;
  194. optional SCMVersionResponseProto versionProto = 4;
  195. optional SendContainerReportProto sendReport = 5;
  196. optional SCMReregisterCmdResponseProto reregisterProto = 6;
  197. optional SCMDeleteBlocksCmdResponseProto deleteBlocksProto = 7;
  198. required string datanodeUUID = 8;
  199. optional SCMCloseContainerCmdResponseProto closeContainerProto = 9;
  200. }
  201. /*
  202. * A group of commands for the datanode to execute
  203. */
  204. message SCMHeartbeatResponseProto {
  205. repeated SCMCommandResponseProto commands = 1;
  206. }
  207. // HB response from SCM, contains a list of block deletion transactions.
  208. message SCMDeleteBlocksCmdResponseProto {
  209. repeated DeletedBlocksTransaction deletedBlocksTransactions = 1;
  210. }
  211. // SendACK response returned by datanode to SCM, currently empty.
  212. message ContainerBlocksDeletionACKResponseProto {
  213. }
  214. // ACK message datanode sent to SCM, contains the result of
  215. // block deletion transactions.
  216. message ContainerBlocksDeletionACKProto {
  217. message DeleteBlockTransactionResult {
  218. required int64 txID = 1;
  219. required bool success = 2;
  220. }
  221. repeated DeleteBlockTransactionResult results = 1;
  222. }
  223. /**
  224. * Protocol used from a datanode to StorageContainerManager.
  225. *
  226. * Please see the request and response messages for details of the RPC calls.
  227. *
  228. * Here is a simple state diagram that shows how a datanode would boot up and
  229. * communicate with SCM.
  230. *
  231. * -----------------------
  232. * | Start |
  233. * ---------- ------------
  234. * |
  235. * |
  236. * |
  237. * |
  238. * |
  239. * |
  240. * |
  241. * ----------v-------------
  242. * | Searching for SCM ------------
  243. * ---------- ------------- |
  244. * | |
  245. * | |
  246. * | ----------v-------------
  247. * | | Register if needed |
  248. * | ----------- ------------
  249. * | |
  250. * v |
  251. * ----------- ---------------- |
  252. * --------- Heartbeat state <--------
  253. * | --------^-------------------
  254. * | |
  255. * | |
  256. * | |
  257. * | |
  258. * | |
  259. * | |
  260. * | |
  261. * ------------------
  262. *
  263. *
  264. *
  265. * Here is how this protocol is used by the datanode. When a datanode boots up
  266. * it moves into a stated called SEARCHING_SCM. In this state datanode is
  267. * trying to establish communication with the SCM. The address of the SCMs are
  268. * retrieved from the configuration information.
  269. *
  270. * In the SEARCHING_SCM state, only rpc call made by datanode is a getVersion
  271. * call to SCM. Once any of the SCMs reply, datanode checks if it has a local
  272. * persisted datanode ID. If it has this means that this datanode is already
  273. * registered with some SCM. If this file is not found, datanode assumes that
  274. * it needs to do a registration.
  275. *
  276. * If registration is need datanode moves into REGISTER state. It will
  277. * send a register call with DatanodeDetailsProto data structure and presist
  278. * that info.
  279. *
  280. * The response to the command contains clusterID. This information is
  281. * also persisted by the datanode and moves into heartbeat state.
  282. *
  283. * Once in the heartbeat state, datanode sends heartbeats and container reports
  284. * to SCM and process commands issued by SCM until it is shutdown.
  285. *
  286. */
  287. service StorageContainerDatanodeProtocolService {
  288. /**
  289. * Gets the version information from the SCM.
  290. */
  291. rpc getVersion (SCMVersionRequestProto) returns (SCMVersionResponseProto);
  292. /**
  293. * Registers a data node with SCM.
  294. */
  295. rpc register (SCMRegisterRequestProto) returns (SCMRegisteredCmdResponseProto);
  296. /**
  297. * Send heartbeat from datanode to SCM. HB's under SCM looks more
  298. * like life line protocol than HB's under HDFS. In other words, it is
  299. * extremely light weight and contains no data payload.
  300. */
  301. rpc sendHeartbeat (SCMHeartbeatRequestProto) returns (SCMHeartbeatResponseProto);
  302. /**
  303. send container reports sends the container report to SCM. This will
  304. return a null command as response.
  305. */
  306. rpc sendContainerReport(ContainerReportsRequestProto) returns (ContainerReportsResponseProto);
  307. /**
  308. * Sends the block deletion ACK to SCM.
  309. */
  310. rpc sendContainerBlocksDeletionACK (ContainerBlocksDeletionACKProto) returns (ContainerBlocksDeletionACKResponseProto);
  311. }