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