JournalProtocol.proto 3.9 KB

123456789101112131415161718192021222324252627282930313233343536373839404142434445464748495051525354555657585960616263646566676869707172737475767778798081828384858687888990919293949596979899100101102103104105106107108109110111112113114115116117118119120121122123124125126127128129
  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. option java_package = "org.apache.hadoop.hdfs.protocol.proto";
  26. option java_outer_classname = "JournalProtocolProtos";
  27. option java_generic_services = true;
  28. option java_generate_equals_and_hash = true;
  29. package hadoop.hdfs;
  30. import "hdfs.proto";
  31. /**
  32. * Journal information used by the journal receiver to identify a journal.
  33. */
  34. message JournalInfoProto {
  35. required string clusterID = 1; // ID of the cluster
  36. optional uint32 layoutVersion = 2; // Layout version
  37. optional uint32 namespaceID = 3; // Namespace ID
  38. }
  39. /**
  40. * journalInfo - the information about the journal
  41. * firstTxnId - the first txid in the journal records
  42. * numTxns - Number of transactions in editlog
  43. * records - bytes containing serialized journal records
  44. * epoch - change to this represents change of journal writer
  45. */
  46. message JournalRequestProto {
  47. required JournalInfoProto journalInfo = 1;
  48. required uint64 firstTxnId = 2;
  49. required uint32 numTxns = 3;
  50. required bytes records = 4;
  51. required uint64 epoch = 5;
  52. }
  53. /**
  54. * void response
  55. */
  56. message JournalResponseProto {
  57. }
  58. /**
  59. * journalInfo - the information about the journal
  60. * txid - first txid in the new log
  61. */
  62. message StartLogSegmentRequestProto {
  63. required JournalInfoProto journalInfo = 1; // Info about the journal
  64. required uint64 txid = 2; // Transaction ID
  65. required uint64 epoch = 3;
  66. }
  67. /**
  68. * void response
  69. */
  70. message StartLogSegmentResponseProto {
  71. }
  72. /**
  73. * journalInfo - the information about the journal
  74. * txid - first txid in the new log
  75. */
  76. message FenceRequestProto {
  77. required JournalInfoProto journalInfo = 1; // Info about the journal
  78. required uint64 epoch = 2; // Epoch - change indicates change in writer
  79. optional string fencerInfo = 3; // Info about fencer for debugging
  80. }
  81. /**
  82. * previousEpoch - previous epoch if any or zero
  83. * lastTransactionId - last valid transaction Id in the journal
  84. * inSync - if all journal segments are available and in sync
  85. */
  86. message FenceResponseProto {
  87. optional uint64 previousEpoch = 1;
  88. optional uint64 lastTransactionId = 2;
  89. optional bool inSync = 3;
  90. }
  91. /**
  92. * Protocol used to journal edits to a remote node. Currently,
  93. * this is used to publish edits from the NameNode to a BackupNode.
  94. *
  95. * See the request and response for details of rpc call.
  96. */
  97. service JournalProtocolService {
  98. /**
  99. * Request sent by active namenode to backup node via
  100. * EditLogBackupOutputStream to stream editlog records.
  101. */
  102. rpc journal(JournalRequestProto) returns (JournalResponseProto);
  103. /**
  104. * Request sent by active namenode to backup node to notify
  105. * that the NameNode has rolled its edit logs and is now writing a
  106. * new log segment.
  107. */
  108. rpc startLogSegment(StartLogSegmentRequestProto)
  109. returns (StartLogSegmentResponseProto);
  110. /**
  111. * Request to fence a journal receiver.
  112. */
  113. rpc fence(FenceRequestProto)
  114. returns (FenceResponseProto);
  115. }