DatanodeProtocol.proto 10 KB

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