DatanodeProtocol.proto 9.7 KB

123456789101112131415161718192021222324252627282930313233343536373839404142434445464748495051525354555657585960616263646566676869707172737475767778798081828384858687888990919293949596979899100101102103104105106107108109110111112113114115116117118119120121122123124125126127128129130131132133134135136137138139140141142143144145146147148149150151152153154155156157158159160161162163164165166167168169170171172173174175176177178179180181182183184185186187188189190191192193194195196197198199200201202203204205206207208209210211212213214215216217218219220221222223224225226227228229230231232233234235236237238239240241242243244245246247248249250251252253254255256257258259260261262263264265266267268269270271272273274275276277278279280281282283284285286287288289290291292293294295296297298299300301302303304305306307308309310311312313314315316317318319320321322323324325326327328329330331332333334335336337338339340341342343344345346347
  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. // This file contains protocol buffers that are used throughout HDFS -- i.e.
  19. // by the client, server, and data transfer protocols.
  20. option java_package = "org.apache.hadoop.hdfs.protocol.proto";
  21. option java_outer_classname = "DatanodeProtocolProtos";
  22. option java_generic_services = true;
  23. option java_generate_equals_and_hash = true;
  24. import "hdfs.proto";
  25. /**
  26. * Information to identify a datanode to a namenode
  27. */
  28. message DatanodeRegistrationProto {
  29. required DatanodeIDProto datanodeID = 1; // Datanode information
  30. required StorageInfoProto storageInfo = 2; // Node information
  31. required ExportedBlockKeysProto keys = 3; // Block keys
  32. }
  33. /**
  34. * Commands sent from namenode to the datanodes
  35. */
  36. message DatanodeCommandProto {
  37. enum Type {
  38. BalancerBandwidthCommand = 0;
  39. BlockCommand = 1;
  40. BlockRecoveryCommand = 2;
  41. FinalizeCommand = 3;
  42. KeyUpdateCommand = 4;
  43. RegisterCommand = 5;
  44. UpgradeCommand = 6;
  45. }
  46. required Type cmdType = 1; // Type of the command
  47. // One of the following command is available when the corresponding
  48. // cmdType is set
  49. optional BalancerBandwidthCommandProto balancerCmd = 2;
  50. optional BlockCommandProto blkCmd = 3;
  51. optional BlockRecoveryCommandProto recoveryCmd = 4;
  52. optional FinalizeCommandProto finalizeCmd = 5;
  53. optional KeyUpdateCommandProto keyUpdateCmd = 6;
  54. optional RegisterCommandProto registerCmd = 7;
  55. optional UpgradeCommandProto upgradeCmd = 8;
  56. }
  57. /**
  58. * Command sent from namenode to datanode to set the
  59. * maximum bandwidth to be used for balancing.
  60. */
  61. message BalancerBandwidthCommandProto {
  62. // Maximum bandwidth to be used by datanode for balancing
  63. required uint64 bandwidth = 1;
  64. }
  65. /**
  66. * Command to instruct datanodes to perform certain action
  67. * on the given set of blocks.
  68. */
  69. message BlockCommandProto {
  70. enum Action {
  71. TRANSFER = 1; // Transfer blocks to another datanode
  72. INVALIDATE = 2; // Invalidate blocks
  73. }
  74. required Action action = 1;
  75. required string blockPoolId = 2;
  76. repeated BlockProto blocks = 3;
  77. repeated DatanodeInfosProto targets = 4;
  78. }
  79. /**
  80. * List of blocks to be recovered by the datanode
  81. */
  82. message BlockRecoveryCommandProto {
  83. repeated RecoveringBlockProto blocks = 1;
  84. }
  85. /**
  86. * Finalize the upgrade at the datanode
  87. */
  88. message FinalizeCommandProto {
  89. required string blockPoolId = 1; // Block pool to be finalized
  90. }
  91. /**
  92. * Update the block keys at the datanode
  93. */
  94. message KeyUpdateCommandProto {
  95. required ExportedBlockKeysProto keys = 1;
  96. }
  97. /**
  98. * Instruct datanode to register with the namenode
  99. */
  100. message RegisterCommandProto {
  101. // void
  102. }
  103. /**
  104. * Generic distributed upgrade Command
  105. */
  106. message UpgradeCommandProto {
  107. enum Action {
  108. UNKNOWN = 0; // Unknown action
  109. REPORT_STATUS = 100; // Report upgrade status
  110. START_UPGRADE = 101; // Start upgrade
  111. }
  112. required Action action = 1; // Upgrade action
  113. required uint32 version = 2; // Version of the upgrade
  114. required uint32 upgradeStatus = 3; // % completed in range 0 & 100
  115. }
  116. /**
  117. * registration - Information of the datanode registering with the namenode
  118. */
  119. message RegisterDatanodeRequestProto {
  120. required DatanodeRegistrationProto registration = 1; // Datanode info
  121. }
  122. /**
  123. * registration - Update registration of the datanode that successfully
  124. * registered. StorageInfo will be updated to include new
  125. * storage ID if the datanode did not have one in the request.
  126. */
  127. message RegisterDatanodeResponseProto {
  128. required DatanodeRegistrationProto registration = 1; // Datanode info
  129. }
  130. /**
  131. * registration - datanode registration information
  132. * capacity - total storage capacity available at the datanode
  133. * dfsUsed - storage used by HDFS
  134. * remaining - remaining storage available for HDFS
  135. * blockPoolUsed - storage used by the block pool
  136. * xmitsInProgress - number of transfers from this datanode to others
  137. * xceiverCount - number of active transceiver threads
  138. * failedVolumes - number of failed volumes
  139. */
  140. message HeartbeatRequestProto {
  141. required DatanodeRegistrationProto registration = 1; // Datanode info
  142. required uint64 capacity = 2;
  143. required uint64 dfsUsed = 3;
  144. required uint64 remaining = 4;
  145. required uint64 blockPoolUsed = 5;
  146. required uint32 xmitsInProgress = 6;
  147. required uint32 xceiverCount = 7;
  148. required uint32 failedVolumes = 8;
  149. }
  150. /**
  151. * cmds - Commands from namenode to datanode.
  152. */
  153. message HeartbeatResponseProto {
  154. repeated DatanodeCommandProto cmds = 1; // Returned commands can be null
  155. }
  156. /**
  157. * registration - datanode registration information
  158. * blockPoolID - block pool ID of the reported blocks
  159. * blocks - each block is represented as two longs in the array.
  160. * first long represents block ID
  161. * second long represents length
  162. */
  163. message BlockReportRequestProto {
  164. required DatanodeRegistrationProto registration = 1;
  165. required string blockPoolId = 2;
  166. repeated uint64 blocks = 3 [packed=true];
  167. }
  168. /**
  169. * cmd - Command from namenode to the datanode
  170. */
  171. message BlockReportResponseProto {
  172. required DatanodeCommandProto cmd = 1;
  173. }
  174. /**
  175. * Data structure to send received or deleted block information
  176. * from datanode to namenode.
  177. *
  178. * deleteHint set to "-" indicates block deletion.
  179. * other deleteHint indicates block addition.
  180. */
  181. message ReceivedDeletedBlockInfoProto {
  182. required BlockProto block = 1;
  183. optional string deleteHint = 2;
  184. }
  185. /**
  186. * registration - datanode registration information
  187. * blockPoolID - block pool ID of the reported blocks
  188. * blocks - Received/deleted block list
  189. */
  190. message BlockReceivedAndDeletedRequestProto {
  191. required DatanodeRegistrationProto registration = 1;
  192. required string blockPoolId = 2;
  193. repeated ReceivedDeletedBlockInfoProto blocks = 3;
  194. }
  195. /**
  196. * void response
  197. */
  198. message BlockReceivedAndDeletedResponseProto {
  199. }
  200. /**
  201. * registartion - Datanode reporting the error
  202. * errorCode - error code indicating the error
  203. * msg - Free text description of the error
  204. */
  205. message ErrorReportRequestProto {
  206. enum ErrorCode {
  207. NOTIFY = 0; // Error report to be logged at the namenode
  208. DISK_ERROR = 1; // DN has disk errors but still has valid volumes
  209. INVALID_BLOCK = 2; // Command from namenode has invalid block ID
  210. FATAL_DISK_ERROR = 3; // No valid volumes left on datanode
  211. }
  212. required DatanodeRegistrationProto registartion = 1; // Registartion info
  213. required uint32 errorCode = 2; // Error code
  214. required string msg = 3; // Error message
  215. }
  216. /**
  217. * void response
  218. */
  219. message ErrorReportResponseProto {
  220. }
  221. /**
  222. * cmd - Upgrade command sent from datanode to namenode
  223. */
  224. message ProcessUpgradeRequestProto {
  225. optional UpgradeCommandProto cmd = 1;
  226. }
  227. /**
  228. * cmd - Upgrade command sent from namenode to datanode
  229. */
  230. message ProcessUpgradeResponseProto {
  231. optional UpgradeCommandProto cmd = 1;
  232. }
  233. /**
  234. * blocks - list of blocks that are reported as corrupt
  235. */
  236. message ReportBadBlocksRequestProto {
  237. repeated LocatedBlockProto blocks = 1;
  238. }
  239. /**
  240. * void response
  241. */
  242. message ReportBadBlocksResponseProto {
  243. }
  244. /**
  245. * Commit block synchronization request during lease recovery
  246. */
  247. message CommitBlockSynchronizationRequestProto {
  248. required ExtendedBlockProto block = 1;
  249. required uint64 newGenStamp = 2;
  250. required uint64 newLength = 3;
  251. required bool closeFile = 4;
  252. required bool deleteBlock = 5;
  253. repeated DatanodeIDProto newTaragets = 6;
  254. }
  255. /**
  256. * void response
  257. */
  258. message CommitBlockSynchronizationResponseProto {
  259. }
  260. /**
  261. * Protocol used from datanode to the namenode
  262. * See the request and response for details of rpc call.
  263. */
  264. service DatanodeProtocolService {
  265. /**
  266. * Register a datanode at a namenode
  267. */
  268. rpc registerDatanode(RegisterDatanodeRequestProto)
  269. returns(RegisterDatanodeResponseProto);
  270. /**
  271. * Send heartbeat from datanode to namenode
  272. */
  273. rpc sendHeartbeat(HeartbeatRequestProto) returns(HeartbeatResponseProto);
  274. /**
  275. * Report blocks at a given datanode to the namenode
  276. */
  277. rpc blockReport(BlockReportRequestProto) returns(BlockReportResponseProto);
  278. /**
  279. * Report from datanode about recently received or deleted block
  280. */
  281. rpc blockReceivedAndDeleted(BlockReceivedAndDeletedRequestProto)
  282. returns(BlockReceivedAndDeletedResponseProto);
  283. /**
  284. * Report from a datanode of an error to the active namenode.
  285. * Used for debugging.
  286. */
  287. rpc errorReport(ErrorReportRequestProto) returns(ErrorReportResponseProto);
  288. /**
  289. * Request the version
  290. */
  291. rpc versionRequest(VersionRequestProto) returns(VersionResponseProto);
  292. /**
  293. * Generic way to send commands from datanode to namenode during
  294. * distributed upgrade process.
  295. */
  296. rpc processUpgrade(ProcessUpgradeRequestProto) returns(ProcessUpgradeResponseProto);
  297. /**
  298. * Report corrupt blocks at the specified location
  299. */
  300. rpc reportBadBlocks(ReportBadBlocksRequestProto) returns(ReportBadBlocksResponseProto);
  301. /**
  302. * Commit block synchronization during lease recovery.
  303. */
  304. rpc commitBlockSynchronization(CommitBlockSynchronizationRequestProto)
  305. returns(CommitBlockSynchronizationResponseProto);
  306. }