DatanodeContainerProtocol.proto 12 KB

123456789101112131415161718192021222324252627282930313233343536373839404142434445464748495051525354555657585960616263646566676869707172737475767778798081828384858687888990919293949596979899100101102103104105106107108109110111112113114115116117118119120121122123124125126127128129130131132133134135136137138139140141142143144145146147148149150151152153154155156157158159160161162163164165166167168169170171172173174175176177178179180181182183184185186187188189190191192193194195196197198199200201202203204205206207208209210211212213214215216217218219220221222223224225226227228229230231232233234235236237238239240241242243244245246247248249250251252253254255256257258259260261262263264265266267268269270271272273274275276277278279280281282283284285286287288289290291292293294295296297298299300301302303304305306307308309310311312313314315316317318319320321322323324325326327328329330331332333334335336337338339340341342343344345346347348349350351352353354355356357358359360361362363364365366367368369370371372373374375376377378379380381382383384385386387388389390391392393394395396397398399400401402403404405406407408409410411412413414415416417418419420421422423424425426427428429430431432433434435436437438439440441442443444445446447448449450451
  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://hadoop.apache.org/docs/stable/hadoop-project-dist/hadoop-common/InterfaceClassification.html
  21. * for what changes are allowed for a *Unstable* .proto interface.
  22. */
  23. // This file contains protocol buffers that are used to transfer data
  24. // to and from the datanode.
  25. syntax = "proto2";
  26. option java_package = "org.apache.hadoop.hdds.protocol.datanode.proto";
  27. option java_outer_classname = "ContainerProtos";
  28. option java_generate_equals_and_hash = true;
  29. package hadoop.hdds.datanode;
  30. /**
  31. * Commands that are used to manipulate the state of containers on a datanode.
  32. *
  33. * These commands allow us to work against the datanode - from
  34. * StorageContainer Manager as well as clients.
  35. *
  36. * 1. CreateContainer - This call is usually made by Storage Container
  37. * manager, when we need to create a new container on a given datanode.
  38. *
  39. * 2. ReadContainer - Allows end user to stat a container. For example
  40. * this allows us to return the metadata of a container.
  41. *
  42. * 3. UpdateContainer - Updates a container metadata.
  43. * 4. DeleteContainer - This call is made to delete a container.
  44. *
  45. * 5. ListContainer - Returns the list of containers on this
  46. * datanode. This will be used by tests and tools.
  47. *
  48. * 6. PutBlock - Given a valid container, creates a block.
  49. *
  50. * 7. GetBlock - Allows user to read the metadata of a block.
  51. *
  52. * 8. DeleteBlock - Deletes a given block.
  53. *
  54. * 9. ListBlock - Returns a list of blocks that are present inside
  55. * a given container.
  56. *
  57. * 10. ReadChunk - Allows us to read a chunk.
  58. *
  59. * 11. DeleteChunk - Delete an unused chunk.
  60. *
  61. * 12. WriteChunk - Allows us to write a chunk
  62. *
  63. * 13. ListChunk - Given a Container/Block returns the list of Chunks.
  64. *
  65. * 14. CompactChunk - Re-writes a chunk based on Offsets.
  66. *
  67. * 15. PutSmallFile - A single RPC that combines both putBlock and WriteChunk.
  68. *
  69. * 16. GetSmallFile - A single RPC that combines both getBlock and ReadChunk.
  70. *
  71. * 17. CloseContainer - Closes an open container and makes it immutable.
  72. *
  73. * 18. CopyContainer - Copies a container from a remote machine.
  74. */
  75. enum Type {
  76. CreateContainer = 1;
  77. ReadContainer = 2;
  78. UpdateContainer = 3;
  79. DeleteContainer = 4;
  80. ListContainer = 5;
  81. PutBlock = 6;
  82. GetBlock = 7;
  83. DeleteBlock = 8;
  84. ListBlock = 9;
  85. ReadChunk = 10;
  86. DeleteChunk = 11;
  87. WriteChunk = 12;
  88. ListChunk = 13;
  89. CompactChunk = 14;
  90. /** Combines Block and Chunk Operation into Single RPC. */
  91. PutSmallFile = 15;
  92. GetSmallFile = 16;
  93. CloseContainer = 17;
  94. GetCommittedBlockLength = 18;
  95. }
  96. enum Result {
  97. SUCCESS = 1;
  98. UNSUPPORTED_REQUEST = 2;
  99. MALFORMED_REQUEST = 3;
  100. CONTAINER_INTERNAL_ERROR = 4;
  101. INVALID_CONFIG = 5;
  102. INVALID_FILE_HASH_FOUND = 6;
  103. CONTAINER_EXISTS = 7;
  104. NO_SUCH_ALGORITHM = 8;
  105. CONTAINER_NOT_FOUND = 9;
  106. IO_EXCEPTION = 10;
  107. UNABLE_TO_READ_METADATA_DB = 11;
  108. NO_SUCH_BLOCK = 12;
  109. OVERWRITE_FLAG_REQUIRED = 13;
  110. UNABLE_TO_FIND_DATA_DIR = 14;
  111. INVALID_WRITE_SIZE = 15;
  112. CHECKSUM_MISMATCH = 16;
  113. UNABLE_TO_FIND_CHUNK = 17;
  114. PROTOC_DECODING_ERROR = 18;
  115. INVALID_ARGUMENT = 19;
  116. PUT_SMALL_FILE_ERROR = 20;
  117. GET_SMALL_FILE_ERROR = 21;
  118. CLOSED_CONTAINER_IO = 22;
  119. ERROR_CONTAINER_NOT_EMPTY = 23;
  120. ERROR_IN_COMPACT_DB = 24;
  121. UNCLOSED_CONTAINER_IO = 25;
  122. DELETE_ON_OPEN_CONTAINER = 26;
  123. CLOSED_CONTAINER_RETRY = 27;
  124. INVALID_CONTAINER_STATE = 28;
  125. DISK_OUT_OF_SPACE = 29;
  126. CONTAINER_ALREADY_EXISTS = 30;
  127. CONTAINER_METADATA_ERROR = 31;
  128. CONTAINER_FILES_CREATE_ERROR = 32;
  129. CONTAINER_CHECKSUM_ERROR = 33;
  130. UNKNOWN_CONTAINER_TYPE = 34;
  131. BLOCK_NOT_COMMITTED = 35;
  132. }
  133. /**
  134. * Block ID that uniquely identify a block in Datanode.
  135. */
  136. message DatanodeBlockID {
  137. required int64 containerID = 1;
  138. required int64 localID = 2;
  139. }
  140. message KeyValue {
  141. required string key = 1;
  142. optional string value = 2;
  143. }
  144. /**
  145. * Lifecycle states of a container in Datanode.
  146. */
  147. enum ContainerLifeCycleState {
  148. OPEN = 1;
  149. CLOSING = 2;
  150. CLOSED = 3;
  151. INVALID = 4;
  152. }
  153. message ContainerCommandRequestProto {
  154. required Type cmdType = 1; // Type of the command
  155. // A string that identifies this command, we generate Trace ID in Ozone
  156. // frontend and this allows us to trace that command all over ozone.
  157. optional string traceID = 2;
  158. required int64 containerID = 3;
  159. required string datanodeUuid = 4;
  160. // One of the following command is available when the corresponding
  161. // cmdType is set. At the protocol level we allow only
  162. // one command in each packet.
  163. // TODO : Upgrade to Protobuf 2.6 or later.
  164. optional CreateContainerRequestProto createContainer = 5;
  165. optional ReadContainerRequestProto readContainer = 6;
  166. optional UpdateContainerRequestProto updateContainer = 7;
  167. optional DeleteContainerRequestProto deleteContainer = 8;
  168. optional ListContainerRequestProto listContainer = 9;
  169. optional CloseContainerRequestProto closeContainer = 10;
  170. optional PutBlockRequestProto putBlock = 11;
  171. optional GetBlockRequestProto getBlock = 12;
  172. optional DeleteBlockRequestProto deleteBlock = 13;
  173. optional ListBlockRequestProto listBlock = 14;
  174. optional ReadChunkRequestProto readChunk = 15;
  175. optional WriteChunkRequestProto writeChunk = 16;
  176. optional DeleteChunkRequestProto deleteChunk = 17;
  177. optional ListChunkRequestProto listChunk = 18;
  178. optional PutSmallFileRequestProto putSmallFile = 19;
  179. optional GetSmallFileRequestProto getSmallFile = 20;
  180. optional GetCommittedBlockLengthRequestProto getCommittedBlockLength = 21;
  181. }
  182. message ContainerCommandResponseProto {
  183. required Type cmdType = 1;
  184. optional string traceID = 2;
  185. required Result result = 3;
  186. optional string message = 4;
  187. optional CreateContainerResponseProto createContainer = 5;
  188. optional ReadContainerResponseProto readContainer = 6;
  189. optional UpdateContainerResponseProto updateContainer = 7;
  190. optional DeleteContainerResponseProto deleteContainer = 8;
  191. optional ListContainerResponseProto listContainer = 9;
  192. optional CloseContainerResponseProto closeContainer = 10;
  193. optional PutBlockResponseProto putBlock = 11;
  194. optional GetBlockResponseProto getBlock = 12;
  195. optional DeleteBlockResponseProto deleteBlock = 13;
  196. optional ListBlockResponseProto listBlock = 14;
  197. optional WriteChunkResponseProto writeChunk = 15;
  198. optional ReadChunkResponseProto readChunk = 16;
  199. optional DeleteChunkResponseProto deleteChunk = 17;
  200. optional ListChunkResponseProto listChunk = 18;
  201. optional PutSmallFileResponseProto putSmallFile = 19;
  202. optional GetSmallFileResponseProto getSmallFile = 20;
  203. optional GetCommittedBlockLengthResponseProto getCommittedBlockLength = 21;
  204. }
  205. message ContainerData {
  206. required int64 containerID = 1;
  207. repeated KeyValue metadata = 2;
  208. optional string containerPath = 4;
  209. optional int64 bytesUsed = 6;
  210. optional int64 size = 7;
  211. optional int64 blockCount = 8;
  212. optional ContainerLifeCycleState state = 9 [default = OPEN];
  213. optional ContainerType containerType = 10 [default = KeyValueContainer];
  214. }
  215. enum ContainerType {
  216. KeyValueContainer = 1;
  217. }
  218. // Container Messages.
  219. message CreateContainerRequestProto {
  220. repeated KeyValue metadata = 2;
  221. optional ContainerType containerType = 3 [default = KeyValueContainer];
  222. }
  223. message CreateContainerResponseProto {
  224. }
  225. message ReadContainerRequestProto {
  226. }
  227. message ReadContainerResponseProto {
  228. optional ContainerData containerData = 1;
  229. }
  230. message UpdateContainerRequestProto {
  231. repeated KeyValue metadata = 2;
  232. optional bool forceUpdate = 3 [default = false];
  233. }
  234. message UpdateContainerResponseProto {
  235. }
  236. message DeleteContainerRequestProto {
  237. optional bool forceDelete = 2 [default = false];
  238. }
  239. message DeleteContainerResponseProto {
  240. }
  241. message ListContainerRequestProto {
  242. optional uint32 count = 2; // Max Results to return
  243. }
  244. message ListContainerResponseProto {
  245. repeated ContainerData containerData = 1;
  246. }
  247. message CloseContainerRequestProto {
  248. }
  249. message CloseContainerResponseProto {
  250. optional string hash = 1;
  251. optional int64 containerID = 2;
  252. }
  253. message BlockData {
  254. required DatanodeBlockID blockID = 1;
  255. optional int64 flags = 2; // for future use.
  256. repeated KeyValue metadata = 3;
  257. repeated ChunkInfo chunks = 4;
  258. optional int64 size = 5;
  259. }
  260. // Block Messages.
  261. message PutBlockRequestProto {
  262. required BlockData blockData = 1;
  263. }
  264. message PutBlockResponseProto {
  265. required GetCommittedBlockLengthResponseProto committedBlockLength = 1;
  266. }
  267. message GetBlockRequestProto {
  268. required DatanodeBlockID blockID = 1;
  269. }
  270. message GetBlockResponseProto {
  271. required BlockData blockData = 1;
  272. }
  273. message DeleteBlockRequestProto {
  274. required DatanodeBlockID blockID = 1;
  275. }
  276. message GetCommittedBlockLengthRequestProto {
  277. required DatanodeBlockID blockID = 1;
  278. }
  279. message GetCommittedBlockLengthResponseProto {
  280. required DatanodeBlockID blockID = 1;
  281. required int64 blockLength = 2;
  282. }
  283. message DeleteBlockResponseProto {
  284. }
  285. message ListBlockRequestProto {
  286. optional int64 startLocalID = 2;
  287. required uint32 count = 3;
  288. }
  289. message ListBlockResponseProto {
  290. repeated BlockData blockData = 1;
  291. }
  292. // Chunk Operations
  293. message ChunkInfo {
  294. required string chunkName = 1;
  295. required uint64 offset = 2;
  296. required uint64 len = 3;
  297. optional string checksum = 4;
  298. repeated KeyValue metadata = 5;
  299. }
  300. enum Stage {
  301. WRITE_DATA = 1;
  302. COMMIT_DATA = 2;
  303. COMBINED = 3;
  304. }
  305. message WriteChunkRequestProto {
  306. required DatanodeBlockID blockID = 1;
  307. required ChunkInfo chunkData = 2;
  308. optional bytes data = 3;
  309. optional Stage stage = 4 [default = COMBINED];
  310. }
  311. message WriteChunkResponseProto {
  312. }
  313. message ReadChunkRequestProto {
  314. required DatanodeBlockID blockID = 1;
  315. required ChunkInfo chunkData = 2;
  316. }
  317. message ReadChunkResponseProto {
  318. required DatanodeBlockID blockID = 1;
  319. required ChunkInfo chunkData = 2;
  320. required bytes data = 3;
  321. }
  322. message DeleteChunkRequestProto {
  323. required DatanodeBlockID blockID = 1;
  324. required ChunkInfo chunkData = 2;
  325. }
  326. message DeleteChunkResponseProto {
  327. }
  328. message ListChunkRequestProto {
  329. required DatanodeBlockID blockID = 1;
  330. required string prevChunkName = 2;
  331. required uint32 count = 3;
  332. }
  333. message ListChunkResponseProto {
  334. repeated ChunkInfo chunkData = 1;
  335. }
  336. /** For small file access combines write chunk and putBlock into a single
  337. RPC */
  338. message PutSmallFileRequestProto {
  339. required PutBlockRequestProto block = 1;
  340. required ChunkInfo chunkInfo = 2;
  341. required bytes data = 3;
  342. }
  343. message PutSmallFileResponseProto {
  344. }
  345. message GetSmallFileRequestProto {
  346. required GetBlockRequestProto block = 1;
  347. }
  348. message GetSmallFileResponseProto {
  349. required ReadChunkResponseProto data = 1;
  350. }
  351. message CopyContainerRequestProto {
  352. required int64 containerID = 1;
  353. required uint64 readOffset = 2;
  354. optional uint64 len = 3;
  355. }
  356. message CopyContainerResponseProto {
  357. required int64 containerID = 1;
  358. required uint64 readOffset = 2;
  359. required uint64 len = 3;
  360. required bool eof = 4;
  361. required bytes data = 5;
  362. optional int64 checksum = 6;
  363. }
  364. service XceiverClientProtocolService {
  365. // A client-to-datanode RPC to send container commands
  366. rpc send(stream ContainerCommandRequestProto) returns
  367. (stream ContainerCommandResponseProto) {};
  368. }
  369. service IntraDatanodeProtocolService {
  370. // An intradatanode service to copy the raw containerdata betwen nodes
  371. rpc download (CopyContainerRequestProto) returns (stream CopyContainerResponseProto);
  372. }