NamenodeProtocol.proto 8.2 KB

123456789101112131415161718192021222324252627282930313233343536373839404142434445464748495051525354555657585960616263646566676869707172737475767778798081828384858687888990919293949596979899100101102103104105106107108109110111112113114115116117118119120121122123124125126127128129130131132133134135136137138139140141142143144145146147148149150151152153154155156157158159160161162163164165166167168169170171172173174175176177178179180181182183184185186187188189190191192193194195196197198199200201202203204205206207208209210211212213214215216217218219220221222223224225226227228229230231232233234235236237238239240241242243244245246247248249250251252253254255256257258259260261262263264265266267268269270271272273274275276277278279280281282283284285286287288289290291292293294295296297298299300301302303304305306307308309310311
  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 = "NamenodeProtocolProtos";
  28. option java_generic_services = true;
  29. option java_generate_equals_and_hash = true;
  30. package hadoop.hdfs.namenode;
  31. import "hdfs.proto";
  32. import "HdfsServer.proto";
  33. /**
  34. * Get list of blocks for a given datanode with the total length
  35. * of adding up to given size
  36. * datanode - Datanode ID to get list of block from
  37. * size - size to which the block lengths must add up to
  38. */
  39. message GetBlocksRequestProto {
  40. required DatanodeIDProto datanode = 1; // Datanode ID
  41. required uint64 size = 2; // Size in bytes
  42. // Minimum Block Size in bytes, adding default value to 10MB, as this might
  43. // cause problem during rolling upgrade, when balancers are upgraded later.
  44. // For more info refer HDFS-13356
  45. optional uint64 minBlockSize = 3 [default = 10485760];
  46. }
  47. /**
  48. * blocks - List of returned blocks
  49. */
  50. message GetBlocksResponseProto {
  51. required BlocksWithLocationsProto blocks = 1; // List of blocks
  52. }
  53. /**
  54. * void request
  55. */
  56. message GetBlockKeysRequestProto {
  57. }
  58. /**
  59. * keys - Information about block keys at the active namenode
  60. */
  61. message GetBlockKeysResponseProto {
  62. optional ExportedBlockKeysProto keys = 1;
  63. }
  64. /**
  65. * void request
  66. */
  67. message GetTransactionIdRequestProto {
  68. }
  69. /**
  70. * txId - Transaction ID of the most recently persisted edit log record
  71. */
  72. message GetTransactionIdResponseProto {
  73. required uint64 txId = 1; // Transaction ID
  74. }
  75. /**
  76. * void request
  77. */
  78. message RollEditLogRequestProto {
  79. }
  80. /**
  81. * signature - A unique token to identify checkpoint transaction
  82. */
  83. message RollEditLogResponseProto {
  84. required CheckpointSignatureProto signature = 1;
  85. }
  86. /**
  87. * void request
  88. */
  89. message GetMostRecentCheckpointTxIdRequestProto {
  90. }
  91. message GetMostRecentCheckpointTxIdResponseProto{
  92. required uint64 txId = 1;
  93. }
  94. /**
  95. * registration - Namenode reporting the error
  96. * errorCode - error code indicating the error
  97. * msg - Free text description of the error
  98. */
  99. message ErrorReportRequestProto {
  100. required NamenodeRegistrationProto registration = 1; // Registration info
  101. required uint32 errorCode = 2; // Error code
  102. required string msg = 3; // Error message
  103. }
  104. /**
  105. * void response
  106. */
  107. message ErrorReportResponseProto {
  108. }
  109. /**
  110. * registration - Information of the namenode registering with primary namenode
  111. */
  112. message RegisterRequestProto {
  113. required NamenodeRegistrationProto registration = 1; // Registration info
  114. }
  115. /**
  116. * registration - Updated registration information of the newly registered
  117. * datanode.
  118. */
  119. message RegisterResponseProto {
  120. required NamenodeRegistrationProto registration = 1; // Registration info
  121. }
  122. /**
  123. * Start checkpoint request
  124. * registration - Namenode that is starting the checkpoint
  125. */
  126. message StartCheckpointRequestProto {
  127. required NamenodeRegistrationProto registration = 1; // Registration info
  128. }
  129. /**
  130. * command - Command returned by the active namenode to be
  131. * be handled by the caller.
  132. */
  133. message StartCheckpointResponseProto {
  134. required NamenodeCommandProto command = 1;
  135. }
  136. /**
  137. * End or finalize the previously started checkpoint
  138. * registration - Namenode that is ending the checkpoint
  139. * signature - unique token to identify checkpoint transaction,
  140. * that was received when checkpoint was started.
  141. */
  142. message EndCheckpointRequestProto {
  143. required NamenodeRegistrationProto registration = 1; // Registration info
  144. required CheckpointSignatureProto signature = 2;
  145. }
  146. /**
  147. * void response
  148. */
  149. message EndCheckpointResponseProto {
  150. }
  151. /**
  152. * sinceTxId - return the editlog information for transactions >= sinceTxId
  153. */
  154. message GetEditLogManifestRequestProto {
  155. required uint64 sinceTxId = 1; // Transaction ID
  156. }
  157. /**
  158. * manifest - Enumeration of editlogs from namenode for
  159. * logs >= sinceTxId in the request
  160. */
  161. message GetEditLogManifestResponseProto {
  162. required RemoteEditLogManifestProto manifest = 1;
  163. }
  164. /**
  165. * void request
  166. */
  167. message IsUpgradeFinalizedRequestProto {
  168. }
  169. message IsUpgradeFinalizedResponseProto {
  170. required bool isUpgradeFinalized = 1;
  171. }
  172. /**
  173. * void request
  174. */
  175. message IsRollingUpgradeRequestProto {
  176. }
  177. message IsRollingUpgradeResponseProto {
  178. required bool isRollingUpgrade = 1;
  179. }
  180. message GetFilePathRequestProto {
  181. required uint64 fileId = 1;
  182. }
  183. message GetFilePathResponseProto {
  184. required string srcPath = 1;
  185. }
  186. message GetNextSPSPathRequestProto {
  187. }
  188. message GetNextSPSPathResponseProto {
  189. optional uint64 spsPath = 1;
  190. }
  191. /**
  192. * Protocol used by the sub-ordinate namenode to send requests
  193. * the active/primary namenode.
  194. *
  195. * See the request and response for details of rpc call.
  196. */
  197. service NamenodeProtocolService {
  198. /**
  199. * Get list of blocks for a given datanode with length
  200. * of blocks adding up to given size.
  201. */
  202. rpc getBlocks(GetBlocksRequestProto) returns(GetBlocksResponseProto);
  203. /**
  204. * Get the current block keys
  205. */
  206. rpc getBlockKeys(GetBlockKeysRequestProto) returns(GetBlockKeysResponseProto);
  207. /**
  208. * Get the transaction ID of the most recently persisted editlog record
  209. */
  210. rpc getTransactionId(GetTransactionIdRequestProto)
  211. returns(GetTransactionIdResponseProto);
  212. /**
  213. * Get the transaction ID of the most recently persisted editlog record
  214. */
  215. rpc getMostRecentCheckpointTxId(GetMostRecentCheckpointTxIdRequestProto)
  216. returns(GetMostRecentCheckpointTxIdResponseProto);
  217. /**
  218. * Close the current editlog and open a new one for checkpointing purposes
  219. */
  220. rpc rollEditLog(RollEditLogRequestProto) returns(RollEditLogResponseProto);
  221. /**
  222. * Request info about the version running on this NameNode
  223. */
  224. rpc versionRequest(VersionRequestProto) returns(VersionResponseProto);
  225. /**
  226. * Report from a sub-ordinate namenode of an error to the active namenode.
  227. * Active namenode may decide to unregister the reporting namenode
  228. * depending on the error.
  229. */
  230. rpc errorReport(ErrorReportRequestProto) returns(ErrorReportResponseProto);
  231. /**
  232. * Request to register a sub-ordinate namenode
  233. */
  234. rpc registerSubordinateNamenode(RegisterRequestProto) returns(RegisterResponseProto);
  235. /**
  236. * Request to start a checkpoint.
  237. */
  238. rpc startCheckpoint(StartCheckpointRequestProto)
  239. returns(StartCheckpointResponseProto);
  240. /**
  241. * End of finalize the previously started checkpoint
  242. */
  243. rpc endCheckpoint(EndCheckpointRequestProto)
  244. returns(EndCheckpointResponseProto);
  245. /**
  246. * Get editlog manifests from the active namenode for all the editlogs
  247. */
  248. rpc getEditLogManifest(GetEditLogManifestRequestProto)
  249. returns(GetEditLogManifestResponseProto);
  250. /**
  251. * Return whether the NameNode is in upgrade state (false) or not (true)
  252. */
  253. rpc isUpgradeFinalized(IsUpgradeFinalizedRequestProto)
  254. returns (IsUpgradeFinalizedResponseProto);
  255. /**
  256. * Return whether the NameNode is in rolling upgrade (true) or not (false).
  257. */
  258. rpc isRollingUpgrade(IsRollingUpgradeRequestProto)
  259. returns (IsRollingUpgradeResponseProto);
  260. /**
  261. * Return the sps path from namenode
  262. */
  263. rpc getNextSPSPath(GetNextSPSPathRequestProto)
  264. returns (GetNextSPSPathResponseProto);
  265. }