DatanodeContainerProtocol.proto 11 KB

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