DatanodeContainerProtocol.proto 11 KB

123456789101112131415161718192021222324252627282930313233343536373839404142434445464748495051525354555657585960616263646566676869707172737475767778798081828384858687888990919293949596979899100101102103104105106107108109110111112113114115116117118119120121122123124125126127128129130131132133134135136137138139140141142143144145146147148149150151152153154155156157158159160161162163164165166167168169170171172173174175176177178179180181182183184185186187188189190191192193194195196197198199200201202203204205206207208209210211212213214215216217218219220221222223224225226227228229230231232233234235236237238239240241242243244245246247248249250251252253254255256257258259260261262263264265266267268269270271272273274275276277278279280281282283284285286287288289290291292293294295296297298299300301302303304305306307308309310311312313314315316317318319320321322323324325326327328329330331332333334335336337338339340341342343344345346347348349350351352353354355356357358359360361362363364365366367368369370371372373374375376377378379380381382383384385386387388389390391392393394395396397398399400401402403404405406407408409410411412413414415416417418419420421422423424425426427428
  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. PutKey - Given a valid container, creates a key.
  49. *
  50. * 7. GetKey - Allows user to read the metadata of a Key.
  51. *
  52. * 8. DeleteKey - Deletes a given key.
  53. *
  54. * 9. ListKey - Returns a list of keys 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/Key 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 putKey and WriteChunk.
  68. *
  69. * 16. GetSmallFile - A single RPC that combines both getKey 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. PutKey = 6;
  82. GetKey = 7;
  83. DeleteKey = 8;
  84. ListKey = 9;
  85. ReadChunk = 10;
  86. DeleteChunk = 11;
  87. WriteChunk = 12;
  88. ListChunk = 13;
  89. CompactChunk = 14;
  90. /** Combines Key and Chunk Operation into Single RPC. */
  91. PutSmallFile = 15;
  92. GetSmallFile = 16;
  93. CloseContainer = 17;
  94. }
  95. enum Result {
  96. SUCCESS = 1;
  97. UNSUPPORTED_REQUEST = 2;
  98. MALFORMED_REQUEST = 3;
  99. CONTAINER_INTERNAL_ERROR = 4;
  100. INVALID_CONFIG = 5;
  101. INVALID_FILE_HASH_FOUND = 6;
  102. CONTAINER_EXISTS = 7;
  103. NO_SUCH_ALGORITHM = 8;
  104. CONTAINER_NOT_FOUND = 9;
  105. IO_EXCEPTION = 10;
  106. UNABLE_TO_READ_METADATA_DB = 11;
  107. NO_SUCH_KEY = 12;
  108. OVERWRITE_FLAG_REQUIRED = 13;
  109. UNABLE_TO_FIND_DATA_DIR = 14;
  110. INVALID_WRITE_SIZE = 15;
  111. CHECKSUM_MISMATCH = 16;
  112. UNABLE_TO_FIND_CHUNK = 17;
  113. PROTOC_DECODING_ERROR = 18;
  114. INVALID_ARGUMENT = 19;
  115. PUT_SMALL_FILE_ERROR = 20;
  116. GET_SMALL_FILE_ERROR = 21;
  117. CLOSED_CONTAINER_IO = 22;
  118. ERROR_CONTAINER_NOT_EMPTY = 23;
  119. ERROR_IN_COMPACT_DB = 24;
  120. UNCLOSED_CONTAINER_IO = 25;
  121. DELETE_ON_OPEN_CONTAINER = 26;
  122. CLOSED_CONTAINER_RETRY = 27;
  123. INVALID_CONTAINER_STATE = 28;
  124. }
  125. /**
  126. * Block ID that uniquely identify a block in Datanode.
  127. */
  128. message DatanodeBlockID {
  129. required int64 containerID = 1;
  130. required int64 localID = 2;
  131. }
  132. message KeyValue {
  133. required string key = 1;
  134. optional string value = 2;
  135. }
  136. /**
  137. * Lifecycle states of a container in Datanode.
  138. */
  139. enum ContainerLifeCycleState {
  140. OPEN = 1;
  141. CLOSING = 2;
  142. CLOSED = 3;
  143. INVALID = 4;
  144. }
  145. message ContainerCommandRequestProto {
  146. required Type cmdType = 1; // Type of the command
  147. // A string that identifies this command, we generate Trace ID in Ozone
  148. // frontend and this allows us to trace that command all over ozone.
  149. optional string traceID = 2;
  150. // One of the following command is available when the corresponding
  151. // cmdType is set. At the protocol level we allow only
  152. // one command in each packet.
  153. // TODO : Upgrade to Protobuf 2.6 or later.
  154. optional CreateContainerRequestProto createContainer = 3;
  155. optional ReadContainerRequestProto readContainer = 4;
  156. optional UpdateContainerRequestProto updateContainer = 5;
  157. optional DeleteContainerRequestProto deleteContainer = 6;
  158. optional ListContainerRequestProto listContainer = 7;
  159. optional PutKeyRequestProto putKey = 8;
  160. optional GetKeyRequestProto getKey = 9;
  161. optional DeleteKeyRequestProto deleteKey = 10;
  162. optional ListKeyRequestProto listKey = 11;
  163. optional ReadChunkRequestProto readChunk = 12;
  164. optional WriteChunkRequestProto writeChunk = 13;
  165. optional DeleteChunkRequestProto deleteChunk = 14;
  166. optional ListChunkRequestProto listChunk = 15;
  167. optional PutSmallFileRequestProto putSmallFile = 16;
  168. optional GetSmallFileRequestProto getSmallFile = 17;
  169. optional CloseContainerRequestProto closeContainer = 18;
  170. required string datanodeUuid = 19;
  171. }
  172. message ContainerCommandResponseProto {
  173. required Type cmdType = 1;
  174. optional string traceID = 2;
  175. optional CreateContainerResponseProto createContainer = 3;
  176. optional ReadContainerResponseProto readContainer = 4;
  177. optional UpdateContainerResponseProto updateContainer = 5;
  178. optional DeleteContainerResponseProto deleteContainer = 6;
  179. optional ListContainerResponseProto listContainer = 7;
  180. optional PutKeyResponseProto putKey = 8;
  181. optional GetKeyResponseProto getKey = 9;
  182. optional DeleteKeyResponseProto deleteKey = 10;
  183. optional ListKeyResponseProto listKey = 11;
  184. optional WriteChunkResponseProto writeChunk = 12;
  185. optional ReadChunkResponseProto readChunk = 13;
  186. optional DeleteChunkResponseProto deleteChunk = 14;
  187. optional ListChunkResponseProto listChunk = 15;
  188. required Result result = 17;
  189. optional string message = 18;
  190. optional PutSmallFileResponseProto putSmallFile = 19;
  191. optional GetSmallFileResponseProto getSmallFile = 20;
  192. optional CloseContainerResponseProto closeContainer = 21;
  193. }
  194. message ContainerData {
  195. required int64 containerID = 1;
  196. repeated KeyValue metadata = 2;
  197. optional string dbPath = 3;
  198. optional string containerPath = 4;
  199. optional int64 bytesUsed = 6;
  200. optional int64 size = 7;
  201. optional int64 keyCount = 8;
  202. optional ContainerLifeCycleState state = 9 [default = OPEN];
  203. optional ContainerType containerType = 10 [default = KeyValueContainer];
  204. optional string containerDBType = 11;
  205. }
  206. enum ContainerType {
  207. KeyValueContainer = 1;
  208. }
  209. // Container Messages.
  210. message CreateContainerRequestProto {
  211. required ContainerData containerData = 1;
  212. }
  213. message CreateContainerResponseProto {
  214. }
  215. message ReadContainerRequestProto {
  216. required int64 containerID = 1;
  217. }
  218. message ReadContainerResponseProto {
  219. optional ContainerData containerData = 1;
  220. }
  221. message UpdateContainerRequestProto {
  222. required ContainerData containerData = 1;
  223. optional bool forceUpdate = 2 [default = false];
  224. }
  225. message UpdateContainerResponseProto {
  226. }
  227. message DeleteContainerRequestProto {
  228. required int64 containerID = 1;
  229. optional bool forceDelete = 2 [default = false];
  230. }
  231. message DeleteContainerResponseProto {
  232. }
  233. message ListContainerRequestProto {
  234. required int64 startContainerID = 1;
  235. optional uint32 count = 2; // Max Results to return
  236. }
  237. message ListContainerResponseProto {
  238. repeated ContainerData containerData = 1;
  239. }
  240. message CloseContainerRequestProto {
  241. required int64 containerID = 1;
  242. }
  243. message CloseContainerResponseProto {
  244. optional string hash = 1;
  245. optional int64 containerID = 2;
  246. }
  247. message KeyData {
  248. required DatanodeBlockID blockID = 1;
  249. optional int64 flags = 2; // for future use.
  250. repeated KeyValue metadata = 3;
  251. repeated ChunkInfo chunks = 4;
  252. }
  253. // Key Messages.
  254. message PutKeyRequestProto {
  255. required KeyData keyData = 1;
  256. }
  257. message PutKeyResponseProto {
  258. }
  259. message GetKeyRequestProto {
  260. required KeyData keyData = 1;
  261. }
  262. message GetKeyResponseProto {
  263. required KeyData keyData = 1;
  264. }
  265. message DeleteKeyRequestProto {
  266. required DatanodeBlockID blockID = 1;
  267. }
  268. message DeleteKeyResponseProto {
  269. }
  270. message ListKeyRequestProto {
  271. required int64 containerID = 1;
  272. optional int64 startLocalID = 2;
  273. required uint32 count = 3;
  274. }
  275. message ListKeyResponseProto {
  276. repeated KeyData keyData = 1;
  277. }
  278. // Chunk Operations
  279. message ChunkInfo {
  280. required string chunkName = 1;
  281. required uint64 offset = 2;
  282. required uint64 len = 3;
  283. optional string checksum = 4;
  284. repeated KeyValue metadata = 5;
  285. }
  286. enum Stage {
  287. WRITE_DATA = 1;
  288. COMMIT_DATA = 2;
  289. COMBINED = 3;
  290. }
  291. message WriteChunkRequestProto {
  292. required DatanodeBlockID blockID = 1;
  293. required ChunkInfo chunkData = 2;
  294. optional bytes data = 3;
  295. optional Stage stage = 4 [default = COMBINED];
  296. }
  297. message WriteChunkResponseProto {
  298. }
  299. message ReadChunkRequestProto {
  300. required DatanodeBlockID blockID = 1;
  301. required ChunkInfo chunkData = 2;
  302. }
  303. message ReadChunkResponseProto {
  304. required DatanodeBlockID blockID = 1;
  305. required ChunkInfo chunkData = 2;
  306. required bytes data = 3;
  307. }
  308. message DeleteChunkRequestProto {
  309. required DatanodeBlockID blockID = 1;
  310. required ChunkInfo chunkData = 2;
  311. }
  312. message DeleteChunkResponseProto {
  313. }
  314. message ListChunkRequestProto {
  315. required DatanodeBlockID blockID = 1;
  316. required string prevChunkName = 2;
  317. required uint32 count = 3;
  318. }
  319. message ListChunkResponseProto {
  320. repeated ChunkInfo chunkData = 1;
  321. }
  322. /** For small file access combines write chunk and putKey into a single
  323. RPC */
  324. message PutSmallFileRequestProto {
  325. required PutKeyRequestProto key = 1;
  326. required ChunkInfo chunkInfo = 2;
  327. required bytes data = 3;
  328. }
  329. message PutSmallFileResponseProto {
  330. }
  331. message GetSmallFileRequestProto {
  332. required GetKeyRequestProto key = 1;
  333. }
  334. message GetSmallFileResponseProto {
  335. required ReadChunkResponseProto data = 1;
  336. }
  337. message CopyContainerRequestProto {
  338. required int64 containerID = 1;
  339. required uint64 readOffset = 2;
  340. optional uint64 len = 3;
  341. }
  342. message CopyContainerResponseProto {
  343. required string archiveName = 1;
  344. required uint64 readOffset = 2;
  345. required uint64 len = 3;
  346. required bool eof = 4;
  347. repeated bytes data = 5;
  348. optional int64 checksum = 6;
  349. }
  350. service XceiverClientProtocolService {
  351. // A client-to-datanode RPC to send container commands
  352. rpc send(stream ContainerCommandRequestProto) returns
  353. (stream ContainerCommandResponseProto) {}
  354. }