DatanodeProtocol.proto 14 KB

123456789101112131415161718192021222324252627282930313233343536373839404142434445464748495051525354555657585960616263646566676869707172737475767778798081828384858687888990919293949596979899100101102103104105106107108109110111112113114115116117118119120121122123124125126127128129130131132133134135136137138139140141142143144145146147148149150151152153154155156157158159160161162163164165166167168169170171172173174175176177178179180181182183184185186187188189190191192193194195196197198199200201202203204205206207208209210211212213214215216217218219220221222223224225226227228229230231232233234235236237238239240241242243244245246247248249250251252253254255256257258259260261262263264265266267268269270271272273274275276277278279280281282283284285286287288289290291292293294295296297298299300301302303304305306307308309310311312313314315316317318319320321322323324325326327328329330331332333334335336337338339340341342343344345346347348349350351352353354355356357358359360361362363364365366367368369370371372373374375376377378379380381382383384385386387388389390391392393394395396397398399400401402403404405406407408409410411412413414415416417418419420421422423424425426427428429430431432433434435436437438439440441442443444445446447448449450451452453454455456457458459460461462463464465
  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 stable.
  20. * Please see http://wiki.apache.org/hadoop/Compatibility
  21. * for what changes are allowed for a *stable* .proto interface.
  22. */
  23. // This file contains protocol buffers that are used throughout HDFS -- i.e.
  24. // by the client, server, and data transfer protocols.
  25. syntax = "proto2";
  26. option java_package = "org.apache.hadoop.hdfs.protocol.proto";
  27. option java_outer_classname = "DatanodeProtocolProtos";
  28. option java_generic_services = true;
  29. option java_generate_equals_and_hash = true;
  30. package hadoop.hdfs.datanode;
  31. import "hdfs.proto";
  32. import "HdfsServer.proto";
  33. /**
  34. * Information to identify a datanode to a namenode
  35. */
  36. message DatanodeRegistrationProto {
  37. required DatanodeIDProto datanodeID = 1; // Datanode information
  38. required StorageInfoProto storageInfo = 2; // Node information
  39. required ExportedBlockKeysProto keys = 3; // Block keys
  40. required string softwareVersion = 4; // Software version of the DN, e.g. "2.0.0"
  41. }
  42. /**
  43. * Commands sent from namenode to the datanodes
  44. */
  45. message DatanodeCommandProto {
  46. enum Type {
  47. BalancerBandwidthCommand = 0;
  48. BlockCommand = 1;
  49. BlockRecoveryCommand = 2;
  50. FinalizeCommand = 3;
  51. KeyUpdateCommand = 4;
  52. RegisterCommand = 5;
  53. UnusedUpgradeCommand = 6;
  54. NullDatanodeCommand = 7;
  55. BlockIdCommand = 8;
  56. }
  57. required Type cmdType = 1; // Type of the command
  58. // One of the following command is available when the corresponding
  59. // cmdType is set
  60. optional BalancerBandwidthCommandProto balancerCmd = 2;
  61. optional BlockCommandProto blkCmd = 3;
  62. optional BlockRecoveryCommandProto recoveryCmd = 4;
  63. optional FinalizeCommandProto finalizeCmd = 5;
  64. optional KeyUpdateCommandProto keyUpdateCmd = 6;
  65. optional RegisterCommandProto registerCmd = 7;
  66. optional BlockIdCommandProto blkIdCmd = 8;
  67. }
  68. /**
  69. * Command sent from namenode to datanode to set the
  70. * maximum bandwidth to be used for balancing.
  71. */
  72. message BalancerBandwidthCommandProto {
  73. // Maximum bandwidth to be used by datanode for balancing
  74. required uint64 bandwidth = 1;
  75. }
  76. /**
  77. * Command to instruct datanodes to perform certain action
  78. * on the given set of blocks.
  79. */
  80. message BlockCommandProto {
  81. enum Action {
  82. TRANSFER = 1; // Transfer blocks to another datanode
  83. INVALIDATE = 2; // Invalidate blocks
  84. SHUTDOWN = 3; // Shutdown the datanode
  85. }
  86. required Action action = 1;
  87. required string blockPoolId = 2;
  88. repeated BlockProto blocks = 3;
  89. repeated DatanodeInfosProto targets = 4;
  90. repeated StorageUuidsProto targetStorageUuids = 5;
  91. repeated StorageTypesProto targetStorageTypes = 6;
  92. }
  93. /**
  94. * Command to instruct datanodes to perform certain action
  95. * on the given set of block IDs.
  96. */
  97. message BlockIdCommandProto {
  98. enum Action {
  99. CACHE = 1;
  100. UNCACHE = 2;
  101. }
  102. required Action action = 1;
  103. required string blockPoolId = 2;
  104. repeated uint64 blockIds = 3 [packed=true];
  105. }
  106. /**
  107. * List of blocks to be recovered by the datanode
  108. */
  109. message BlockRecoveryCommandProto {
  110. repeated RecoveringBlockProto blocks = 1;
  111. }
  112. /**
  113. * Finalize the upgrade at the datanode
  114. */
  115. message FinalizeCommandProto {
  116. required string blockPoolId = 1; // Block pool to be finalized
  117. }
  118. /**
  119. * Update the block keys at the datanode
  120. */
  121. message KeyUpdateCommandProto {
  122. required ExportedBlockKeysProto keys = 1;
  123. }
  124. /**
  125. * Instruct datanode to register with the namenode
  126. */
  127. message RegisterCommandProto {
  128. // void
  129. }
  130. /**
  131. * registration - Information of the datanode registering with the namenode
  132. */
  133. message RegisterDatanodeRequestProto {
  134. required DatanodeRegistrationProto registration = 1; // Datanode info
  135. }
  136. /**
  137. * registration - Update registration of the datanode that successfully
  138. * registered. StorageInfo will be updated to include new
  139. * storage ID if the datanode did not have one in the request.
  140. */
  141. message RegisterDatanodeResponseProto {
  142. required DatanodeRegistrationProto registration = 1; // Datanode info
  143. }
  144. /**
  145. * failedStorageLocations - storage locations that have failed
  146. * lastVolumeFailureDate - date/time of last volume failure
  147. * estimatedCapacityLost - estimate of total capacity lost due to volume failures
  148. */
  149. message VolumeFailureSummaryProto {
  150. repeated string failedStorageLocations = 1;
  151. required uint64 lastVolumeFailureDate = 2;
  152. required uint64 estimatedCapacityLostTotal = 3;
  153. }
  154. /**
  155. * registration - datanode registration information
  156. * capacity - total storage capacity available at the datanode
  157. * dfsUsed - storage used by HDFS
  158. * remaining - remaining storage available for HDFS
  159. * blockPoolUsed - storage used by the block pool
  160. * xmitsInProgress - number of transfers from this datanode to others
  161. * xceiverCount - number of active transceiver threads
  162. * failedVolumes - number of failed volumes. This is redundant with the
  163. * information included in volumeFailureSummary, but the field is retained
  164. * for backwards compatibility.
  165. * cacheCapacity - total cache capacity available at the datanode
  166. * cacheUsed - amount of cache used
  167. * volumeFailureSummary - info about volume failures
  168. * slowPeers - info about peer DataNodes that are suspected to be slow.
  169. * slowDisks - info about DataNode disks that are suspected to be slow.
  170. */
  171. message HeartbeatRequestProto {
  172. required DatanodeRegistrationProto registration = 1; // Datanode info
  173. repeated StorageReportProto reports = 2;
  174. optional uint32 xmitsInProgress = 3 [ default = 0 ];
  175. optional uint32 xceiverCount = 4 [ default = 0 ];
  176. optional uint32 failedVolumes = 5 [ default = 0 ];
  177. optional uint64 cacheCapacity = 6 [ default = 0 ];
  178. optional uint64 cacheUsed = 7 [default = 0 ];
  179. optional VolumeFailureSummaryProto volumeFailureSummary = 8;
  180. optional bool requestFullBlockReportLease = 9 [ default = false ];
  181. repeated SlowPeerReportProto slowPeers = 10;
  182. repeated SlowDiskReportProto slowDisks = 11;
  183. }
  184. /**
  185. * cmds - Commands from namenode to datanode.
  186. * haStatus - Status (from an HA perspective) of the NN sending this response
  187. */
  188. message HeartbeatResponseProto {
  189. repeated DatanodeCommandProto cmds = 1; // Returned commands can be null
  190. required NNHAStatusHeartbeatProto haStatus = 2;
  191. optional RollingUpgradeStatusProto rollingUpgradeStatus = 3;
  192. optional RollingUpgradeStatusProto rollingUpgradeStatusV2 = 4;
  193. optional uint64 fullBlockReportLeaseId = 5 [ default = 0 ];
  194. }
  195. /**
  196. * registration - datanode registration information
  197. * blockPoolID - block pool ID of the reported blocks
  198. * blocks - each block is represented as multiple longs in the array.
  199. * first long represents block ID
  200. * second long represents length
  201. * third long represents gen stamp
  202. * fourth long (if under construction) represents replica state
  203. * context - An optional field containing information about the context
  204. * of this block report.
  205. */
  206. message BlockReportRequestProto {
  207. required DatanodeRegistrationProto registration = 1;
  208. required string blockPoolId = 2;
  209. repeated StorageBlockReportProto reports = 3;
  210. optional BlockReportContextProto context = 4;
  211. }
  212. message BlockReportContextProto {
  213. // The total number of RPCs this block report is broken into.
  214. required int32 totalRpcs = 1;
  215. // The index of the current RPC (zero-based)
  216. required int32 curRpc = 2;
  217. // The unique 64-bit ID of this block report
  218. required int64 id = 3;
  219. // The block report lease ID, or 0 if we are sending without a lease to
  220. // bypass rate-limiting.
  221. optional uint64 leaseId = 4 [ default = 0 ];
  222. }
  223. /**
  224. * Report of blocks in a storage
  225. */
  226. message StorageBlockReportProto {
  227. required DatanodeStorageProto storage = 1; // Storage
  228. repeated uint64 blocks = 2 [packed=true];
  229. optional uint64 numberOfBlocks = 3;
  230. repeated bytes blocksBuffers = 4;
  231. }
  232. /**
  233. * cmd - Command from namenode to the datanode
  234. */
  235. message BlockReportResponseProto {
  236. optional DatanodeCommandProto cmd = 1;
  237. }
  238. /**
  239. * registration - datanode registration information
  240. * blockPoolId - block pool ID of the reported blocks
  241. * blocks - representation of blocks as longs for efficiency reasons
  242. */
  243. message CacheReportRequestProto {
  244. required DatanodeRegistrationProto registration = 1;
  245. required string blockPoolId = 2;
  246. repeated uint64 blocks = 3 [packed=true];
  247. }
  248. message CacheReportResponseProto {
  249. optional DatanodeCommandProto cmd = 1;
  250. }
  251. /**
  252. * Data structure to send received or deleted block information
  253. * from datanode to namenode.
  254. */
  255. message ReceivedDeletedBlockInfoProto {
  256. enum BlockStatus {
  257. RECEIVING = 1; // block being created
  258. RECEIVED = 2; // block creation complete
  259. DELETED = 3;
  260. }
  261. required BlockProto block = 1;
  262. required BlockStatus status = 3;
  263. optional string deleteHint = 2;
  264. }
  265. /**
  266. * List of blocks received and deleted for a storage.
  267. */
  268. message StorageReceivedDeletedBlocksProto {
  269. required string storageUuid = 1 [ deprecated = true ];
  270. repeated ReceivedDeletedBlockInfoProto blocks = 2;
  271. optional DatanodeStorageProto storage = 3; // supersedes storageUuid.
  272. }
  273. /**
  274. * registration - datanode registration information
  275. * blockPoolID - block pool ID of the reported blocks
  276. * blocks - Received/deleted block list
  277. */
  278. message BlockReceivedAndDeletedRequestProto {
  279. required DatanodeRegistrationProto registration = 1;
  280. required string blockPoolId = 2;
  281. repeated StorageReceivedDeletedBlocksProto blocks = 3;
  282. }
  283. /**
  284. * void response
  285. */
  286. message BlockReceivedAndDeletedResponseProto {
  287. }
  288. /**
  289. * registartion - Datanode reporting the error
  290. * errorCode - error code indicating the error
  291. * msg - Free text description of the error
  292. */
  293. message ErrorReportRequestProto {
  294. enum ErrorCode {
  295. NOTIFY = 0; // Error report to be logged at the namenode
  296. DISK_ERROR = 1; // DN has disk errors but still has valid volumes
  297. INVALID_BLOCK = 2; // Command from namenode has invalid block ID
  298. FATAL_DISK_ERROR = 3; // No valid volumes left on datanode
  299. }
  300. required DatanodeRegistrationProto registartion = 1; // Registartion info
  301. required uint32 errorCode = 2; // Error code
  302. required string msg = 3; // Error message
  303. }
  304. /**
  305. * void response
  306. */
  307. message ErrorReportResponseProto {
  308. }
  309. /**
  310. * blocks - list of blocks that are reported as corrupt
  311. */
  312. message ReportBadBlocksRequestProto {
  313. repeated LocatedBlockProto blocks = 1;
  314. }
  315. /**
  316. * void response
  317. */
  318. message ReportBadBlocksResponseProto {
  319. }
  320. /**
  321. * Commit block synchronization request during lease recovery
  322. */
  323. message CommitBlockSynchronizationRequestProto {
  324. required ExtendedBlockProto block = 1;
  325. required uint64 newGenStamp = 2;
  326. required uint64 newLength = 3;
  327. required bool closeFile = 4;
  328. required bool deleteBlock = 5;
  329. repeated DatanodeIDProto newTaragets = 6;
  330. repeated string newTargetStorages = 7;
  331. }
  332. /**
  333. * void response
  334. */
  335. message CommitBlockSynchronizationResponseProto {
  336. }
  337. /**
  338. * Information about a single slow peer that may be reported by
  339. * the DataNode to the NameNode as part of the heartbeat request.
  340. * The message includes the peer's DataNodeId and its
  341. * aggregate packet latency as observed by the reporting DataNode.
  342. * (DataNodeId must be transmitted as a string for protocol compability
  343. * with earlier versions of Hadoop).
  344. *
  345. * The exact choice of the aggregate is opaque to the NameNode but it
  346. * _should_ be chosen consistenly by all DataNodes in the cluster.
  347. * Examples of aggregates are 90th percentile (good) and mean (not so
  348. * good).
  349. */
  350. message SlowPeerReportProto {
  351. optional string dataNodeId = 1;
  352. optional double aggregateLatency = 2;
  353. }
  354. /**
  355. * Information about a single slow disk that may be reported by
  356. * the DataNode to the NameNode as part of the heartbeat request.
  357. * The message includes the disk's basePath, mean metadata op latency,
  358. * mean read io latency and mean write io latency as observed by the DataNode.
  359. */
  360. message SlowDiskReportProto {
  361. optional string basePath = 1;
  362. optional double meanMetadataOpLatency = 2;
  363. optional double meanReadIoLatency = 3;
  364. optional double meanWriteIoLatency = 4;
  365. }
  366. /**
  367. * Protocol used from datanode to the namenode
  368. * See the request and response for details of rpc call.
  369. */
  370. service DatanodeProtocolService {
  371. /**
  372. * Register a datanode at a namenode
  373. */
  374. rpc registerDatanode(RegisterDatanodeRequestProto)
  375. returns(RegisterDatanodeResponseProto);
  376. /**
  377. * Send heartbeat from datanode to namenode
  378. */
  379. rpc sendHeartbeat(HeartbeatRequestProto) returns(HeartbeatResponseProto);
  380. /**
  381. * Report blocks at a given datanode to the namenode
  382. */
  383. rpc blockReport(BlockReportRequestProto) returns(BlockReportResponseProto);
  384. /**
  385. * Report cached blocks at a datanode to the namenode
  386. */
  387. rpc cacheReport(CacheReportRequestProto) returns(CacheReportResponseProto);
  388. /**
  389. * Incremental block report from the DN. This contains info about recently
  390. * received and deleted blocks, as well as when blocks start being
  391. * received.
  392. */
  393. rpc blockReceivedAndDeleted(BlockReceivedAndDeletedRequestProto)
  394. returns(BlockReceivedAndDeletedResponseProto);
  395. /**
  396. * Report from a datanode of an error to the active namenode.
  397. * Used for debugging.
  398. */
  399. rpc errorReport(ErrorReportRequestProto) returns(ErrorReportResponseProto);
  400. /**
  401. * Request the version
  402. */
  403. rpc versionRequest(VersionRequestProto) returns(VersionResponseProto);
  404. /**
  405. * Report corrupt blocks at the specified location
  406. */
  407. rpc reportBadBlocks(ReportBadBlocksRequestProto) returns(ReportBadBlocksResponseProto);
  408. /**
  409. * Commit block synchronization during lease recovery.
  410. */
  411. rpc commitBlockSynchronization(CommitBlockSynchronizationRequestProto)
  412. returns(CommitBlockSynchronizationResponseProto);
  413. }