QJournalProtocol.proto 8.0 KB

123456789101112131415161718192021222324252627282930313233343536373839404142434445464748495051525354555657585960616263646566676869707172737475767778798081828384858687888990919293949596979899100101102103104105106107108109110111112113114115116117118119120121122123124125126127128129130131132133134135136137138139140141142143144145146147148149150151152153154155156157158159160161162163164165166167168169170171172173174175176177178179180181182183184185186187188189190191192193194195196197198199200201202203204205206207208209210211212213214215216217218219220221222223224225226227228229230231232233234235236237238239240241242243244245246247248249250251252253254255256257258259260261262263264265266267268269270271272273274275276277278279280281282283284285286287288289290291292293294295296297298299300301302303304305306307308309310311312313314315316317318319320321322323324325326327328329330331332333334335336337338339340341342343344
  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. option java_package = "org.apache.hadoop.hdfs.qjournal.protocol";
  24. option java_outer_classname = "QJournalProtocolProtos";
  25. option java_generic_services = true;
  26. option java_generate_equals_and_hash = true;
  27. package hadoop.hdfs;
  28. import "hdfs.proto";
  29. message JournalIdProto {
  30. required string identifier = 1;
  31. }
  32. message RequestInfoProto {
  33. required JournalIdProto journalId = 1;
  34. required uint64 epoch = 2;
  35. required uint64 ipcSerialNumber = 3;
  36. // Whenever a writer makes a request, it informs
  37. // the node of the latest committed txid. This may
  38. // be higher than the transaction data included in the
  39. // request itself, eg in the case that the node has
  40. // fallen behind.
  41. optional uint64 committedTxId = 4;
  42. }
  43. message SegmentStateProto {
  44. required uint64 startTxId = 1;
  45. required uint64 endTxId = 2;
  46. required bool isInProgress = 3;
  47. }
  48. /**
  49. * The storage format used on local disk for previously
  50. * accepted decisions.
  51. */
  52. message PersistedRecoveryPaxosData {
  53. required SegmentStateProto segmentState = 1;
  54. required uint64 acceptedInEpoch = 2;
  55. }
  56. /**
  57. * journal()
  58. */
  59. message JournalRequestProto {
  60. required RequestInfoProto reqInfo = 1;
  61. required uint64 firstTxnId = 2;
  62. required uint32 numTxns = 3;
  63. required bytes records = 4;
  64. required uint64 segmentTxnId = 5;
  65. }
  66. message JournalResponseProto {
  67. }
  68. /**
  69. * heartbeat()
  70. */
  71. message HeartbeatRequestProto {
  72. required RequestInfoProto reqInfo = 1;
  73. }
  74. message HeartbeatResponseProto { // void response
  75. }
  76. /**
  77. * startLogSegment()
  78. */
  79. message StartLogSegmentRequestProto {
  80. required RequestInfoProto reqInfo = 1;
  81. required uint64 txid = 2; // Transaction ID
  82. }
  83. message StartLogSegmentResponseProto {
  84. }
  85. /**
  86. * finalizeLogSegment()
  87. */
  88. message FinalizeLogSegmentRequestProto {
  89. required RequestInfoProto reqInfo = 1;
  90. required uint64 startTxId = 2;
  91. required uint64 endTxId = 3;
  92. }
  93. message FinalizeLogSegmentResponseProto {
  94. }
  95. /**
  96. * purgeLogs()
  97. */
  98. message PurgeLogsRequestProto {
  99. required RequestInfoProto reqInfo = 1;
  100. required uint64 minTxIdToKeep = 2;
  101. }
  102. message PurgeLogsResponseProto {
  103. }
  104. /**
  105. * isFormatted()
  106. */
  107. message IsFormattedRequestProto {
  108. required JournalIdProto jid = 1;
  109. }
  110. message IsFormattedResponseProto {
  111. required bool isFormatted = 1;
  112. }
  113. /**
  114. * getJournalCTime()
  115. */
  116. message GetJournalCTimeRequestProto {
  117. required JournalIdProto jid = 1;
  118. }
  119. message GetJournalCTimeResponseProto {
  120. required int64 resultCTime = 1;
  121. }
  122. /**
  123. * doPreUpgrade()
  124. */
  125. message DoPreUpgradeRequestProto {
  126. required JournalIdProto jid = 1;
  127. }
  128. message DoPreUpgradeResponseProto {
  129. }
  130. /**
  131. * doUpgrade()
  132. */
  133. message DoUpgradeRequestProto {
  134. required JournalIdProto jid = 1;
  135. required StorageInfoProto sInfo = 2;
  136. }
  137. message DoUpgradeResponseProto {
  138. }
  139. /**
  140. * doFinalize()
  141. */
  142. message DoFinalizeRequestProto {
  143. required JournalIdProto jid = 1;
  144. }
  145. message DoFinalizeResponseProto {
  146. }
  147. /**
  148. * canRollBack()
  149. */
  150. message CanRollBackRequestProto {
  151. required JournalIdProto jid = 1;
  152. required StorageInfoProto storage = 2;
  153. required StorageInfoProto prevStorage = 3;
  154. required int32 targetLayoutVersion = 4;
  155. }
  156. message CanRollBackResponseProto {
  157. required bool canRollBack = 1;
  158. }
  159. /**
  160. * doRollback()
  161. */
  162. message DoRollbackRequestProto {
  163. required JournalIdProto jid = 1;
  164. }
  165. message DoRollbackResponseProto {
  166. }
  167. /**
  168. * getJournalState()
  169. */
  170. message GetJournalStateRequestProto {
  171. required JournalIdProto jid = 1;
  172. }
  173. message GetJournalStateResponseProto {
  174. required uint64 lastPromisedEpoch = 1;
  175. // Deprecated by fromURL
  176. required uint32 httpPort = 2;
  177. optional string fromURL = 3;
  178. }
  179. /**
  180. * format()
  181. */
  182. message FormatRequestProto {
  183. required JournalIdProto jid = 1;
  184. required NamespaceInfoProto nsInfo = 2;
  185. }
  186. message FormatResponseProto {
  187. }
  188. /**
  189. * newEpoch()
  190. */
  191. message NewEpochRequestProto {
  192. required JournalIdProto jid = 1;
  193. required NamespaceInfoProto nsInfo = 2;
  194. required uint64 epoch = 3;
  195. }
  196. message NewEpochResponseProto {
  197. optional uint64 lastSegmentTxId = 1;
  198. }
  199. /**
  200. * getEditLogManifest()
  201. */
  202. message GetEditLogManifestRequestProto {
  203. required JournalIdProto jid = 1;
  204. required uint64 sinceTxId = 2; // Transaction ID
  205. // Whether or not the client will be reading from the returned streams.
  206. // optional bool forReading = 3 [default = true]; <obsolete, do not reuse>
  207. optional bool inProgressOk = 4 [default = false];
  208. }
  209. message GetEditLogManifestResponseProto {
  210. required RemoteEditLogManifestProto manifest = 1;
  211. // Deprecated by fromURL
  212. required uint32 httpPort = 2;
  213. optional string fromURL = 3;
  214. // TODO: we should add nsinfo somewhere
  215. // to verify that it matches up with our expectation
  216. // required NamespaceInfoProto nsInfo = 2;
  217. }
  218. /**
  219. * prepareRecovery()
  220. */
  221. message PrepareRecoveryRequestProto {
  222. required RequestInfoProto reqInfo = 1;
  223. required uint64 segmentTxId = 2;
  224. }
  225. message PrepareRecoveryResponseProto {
  226. optional SegmentStateProto segmentState = 1;
  227. optional uint64 acceptedInEpoch = 2;
  228. required uint64 lastWriterEpoch = 3;
  229. // The highest committed txid that this logger has ever seen.
  230. // This may be higher than the data it actually has, in the case
  231. // that it was lagging before the old writer crashed.
  232. optional uint64 lastCommittedTxId = 4;
  233. }
  234. /**
  235. * acceptRecovery()
  236. */
  237. message AcceptRecoveryRequestProto {
  238. required RequestInfoProto reqInfo = 1;
  239. /** Details on the segment to recover */
  240. required SegmentStateProto stateToAccept = 2;
  241. /** The URL from which the log may be copied */
  242. required string fromURL = 3;
  243. }
  244. message AcceptRecoveryResponseProto {
  245. }
  246. /**
  247. * Protocol used to journal edits to a JournalNode.
  248. * See the request and response for details of rpc call.
  249. */
  250. service QJournalProtocolService {
  251. rpc isFormatted(IsFormattedRequestProto) returns (IsFormattedResponseProto);
  252. rpc getJournalCTime(GetJournalCTimeRequestProto) returns (GetJournalCTimeResponseProto);
  253. rpc doPreUpgrade(DoPreUpgradeRequestProto) returns (DoPreUpgradeResponseProto);
  254. rpc doUpgrade(DoUpgradeRequestProto) returns (DoUpgradeResponseProto);
  255. rpc doFinalize(DoFinalizeRequestProto) returns (DoFinalizeResponseProto);
  256. rpc canRollBack(CanRollBackRequestProto) returns (CanRollBackResponseProto);
  257. rpc doRollback(DoRollbackRequestProto) returns (DoRollbackResponseProto);
  258. rpc getJournalState(GetJournalStateRequestProto) returns (GetJournalStateResponseProto);
  259. rpc newEpoch(NewEpochRequestProto) returns (NewEpochResponseProto);
  260. rpc format(FormatRequestProto) returns (FormatResponseProto);
  261. rpc journal(JournalRequestProto) returns (JournalResponseProto);
  262. rpc heartbeat(HeartbeatRequestProto) returns (HeartbeatResponseProto);
  263. rpc startLogSegment(StartLogSegmentRequestProto)
  264. returns (StartLogSegmentResponseProto);
  265. rpc finalizeLogSegment(FinalizeLogSegmentRequestProto)
  266. returns (FinalizeLogSegmentResponseProto);
  267. rpc purgeLogs(PurgeLogsRequestProto)
  268. returns (PurgeLogsResponseProto);
  269. rpc getEditLogManifest(GetEditLogManifestRequestProto)
  270. returns (GetEditLogManifestResponseProto);
  271. rpc prepareRecovery(PrepareRecoveryRequestProto)
  272. returns (PrepareRecoveryResponseProto);
  273. rpc acceptRecovery(AcceptRecoveryRequestProto)
  274. returns (AcceptRecoveryResponseProto);
  275. }