JournalProtocol.proto 4.0 KB

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