123456789101112131415161718192021222324252627282930313233343536373839404142434445464748495051525354555657585960616263646566676869707172737475767778798081828384858687888990919293949596979899100101102103104105106107108109110111112113114115116117118119120121122123124125126127128129130131132133134135136137138139140141142143144145146147148149150151152153154155156157158159160161162163164165166167168169170171172173174175176177178179180181182183184185186187188189190191192193194195196197198199200201202203204205206207208209210211212213214215216217218219220221222223224225226227228229230231232233234235236237238239240241242243244245246247248249250251252253254255256257258259260261262263264265266267268269270271272273274275276277278279280281282283284285286287288289290291292293294295296297298299300301302303304305306307308309310311312313314315316317318319320321322323324325326327328329330331332333334335336337338339340341342343344 |
- /**
- * Licensed to the Apache Software Foundation (ASF) under one
- * or more contributor license agreements. See the NOTICE file
- * distributed with this work for additional information
- * regarding copyright ownership. The ASF licenses this file
- * to you under the Apache License, Version 2.0 (the
- * "License"); you may not use this file except in compliance
- * with the License. You may obtain a copy of the License at
- *
- * http://www.apache.org/licenses/LICENSE-2.0
- *
- * Unless required by applicable law or agreed to in writing, software
- * distributed under the License is distributed on an "AS IS" BASIS,
- * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
- * See the License for the specific language governing permissions and
- * limitations under the License.
- */
- /**
- * These .proto interfaces are private and stable.
- * Please see http://wiki.apache.org/hadoop/Compatibility
- * for what changes are allowed for a *stable* .proto interface.
- */
- option java_package = "org.apache.hadoop.hdfs.qjournal.protocol";
- option java_outer_classname = "QJournalProtocolProtos";
- option java_generic_services = true;
- option java_generate_equals_and_hash = true;
- package hadoop.hdfs;
- import "hdfs.proto";
- message JournalIdProto {
- required string identifier = 1;
- }
- message RequestInfoProto {
- required JournalIdProto journalId = 1;
- required uint64 epoch = 2;
- required uint64 ipcSerialNumber = 3;
- // Whenever a writer makes a request, it informs
- // the node of the latest committed txid. This may
- // be higher than the transaction data included in the
- // request itself, eg in the case that the node has
- // fallen behind.
- optional uint64 committedTxId = 4;
- }
- message SegmentStateProto {
- required uint64 startTxId = 1;
- required uint64 endTxId = 2;
- required bool isInProgress = 3;
- }
- /**
- * The storage format used on local disk for previously
- * accepted decisions.
- */
- message PersistedRecoveryPaxosData {
- required SegmentStateProto segmentState = 1;
- required uint64 acceptedInEpoch = 2;
- }
- /**
- * journal()
- */
- message JournalRequestProto {
- required RequestInfoProto reqInfo = 1;
- required uint64 firstTxnId = 2;
- required uint32 numTxns = 3;
- required bytes records = 4;
- required uint64 segmentTxnId = 5;
- }
- message JournalResponseProto {
- }
- /**
- * heartbeat()
- */
- message HeartbeatRequestProto {
- required RequestInfoProto reqInfo = 1;
- }
- message HeartbeatResponseProto { // void response
- }
- /**
- * startLogSegment()
- */
- message StartLogSegmentRequestProto {
- required RequestInfoProto reqInfo = 1;
- required uint64 txid = 2; // Transaction ID
- }
- message StartLogSegmentResponseProto {
- }
- /**
- * finalizeLogSegment()
- */
- message FinalizeLogSegmentRequestProto {
- required RequestInfoProto reqInfo = 1;
- required uint64 startTxId = 2;
- required uint64 endTxId = 3;
- }
- message FinalizeLogSegmentResponseProto {
- }
- /**
- * purgeLogs()
- */
- message PurgeLogsRequestProto {
- required RequestInfoProto reqInfo = 1;
- required uint64 minTxIdToKeep = 2;
- }
- message PurgeLogsResponseProto {
- }
- /**
- * isFormatted()
- */
- message IsFormattedRequestProto {
- required JournalIdProto jid = 1;
- }
- message IsFormattedResponseProto {
- required bool isFormatted = 1;
- }
- /**
- * getJournalCTime()
- */
- message GetJournalCTimeRequestProto {
- required JournalIdProto jid = 1;
- }
- message GetJournalCTimeResponseProto {
- required int64 resultCTime = 1;
- }
- /**
- * doPreUpgrade()
- */
- message DoPreUpgradeRequestProto {
- required JournalIdProto jid = 1;
- }
- message DoPreUpgradeResponseProto {
- }
- /**
- * doUpgrade()
- */
- message DoUpgradeRequestProto {
- required JournalIdProto jid = 1;
- required StorageInfoProto sInfo = 2;
- }
- message DoUpgradeResponseProto {
- }
- /**
- * doFinalize()
- */
- message DoFinalizeRequestProto {
- required JournalIdProto jid = 1;
- }
- message DoFinalizeResponseProto {
- }
- /**
- * canRollBack()
- */
- message CanRollBackRequestProto {
- required JournalIdProto jid = 1;
- required StorageInfoProto storage = 2;
- required StorageInfoProto prevStorage = 3;
- required int32 targetLayoutVersion = 4;
- }
- message CanRollBackResponseProto {
- required bool canRollBack = 1;
- }
- /**
- * doRollback()
- */
- message DoRollbackRequestProto {
- required JournalIdProto jid = 1;
- }
- message DoRollbackResponseProto {
- }
- /**
- * getJournalState()
- */
- message GetJournalStateRequestProto {
- required JournalIdProto jid = 1;
- }
- message GetJournalStateResponseProto {
- required uint64 lastPromisedEpoch = 1;
- // Deprecated by fromURL
- required uint32 httpPort = 2;
- optional string fromURL = 3;
- }
- /**
- * format()
- */
- message FormatRequestProto {
- required JournalIdProto jid = 1;
- required NamespaceInfoProto nsInfo = 2;
- }
- message FormatResponseProto {
- }
- /**
- * newEpoch()
- */
- message NewEpochRequestProto {
- required JournalIdProto jid = 1;
- required NamespaceInfoProto nsInfo = 2;
- required uint64 epoch = 3;
- }
- message NewEpochResponseProto {
- optional uint64 lastSegmentTxId = 1;
- }
- /**
- * getEditLogManifest()
- */
- message GetEditLogManifestRequestProto {
- required JournalIdProto jid = 1;
- required uint64 sinceTxId = 2; // Transaction ID
- // Whether or not the client will be reading from the returned streams.
- // optional bool forReading = 3 [default = true]; <obsolete, do not reuse>
- optional bool inProgressOk = 4 [default = false];
- }
- message GetEditLogManifestResponseProto {
- required RemoteEditLogManifestProto manifest = 1;
- // Deprecated by fromURL
- required uint32 httpPort = 2;
- optional string fromURL = 3;
- // TODO: we should add nsinfo somewhere
- // to verify that it matches up with our expectation
- // required NamespaceInfoProto nsInfo = 2;
- }
- /**
- * prepareRecovery()
- */
- message PrepareRecoveryRequestProto {
- required RequestInfoProto reqInfo = 1;
- required uint64 segmentTxId = 2;
- }
- message PrepareRecoveryResponseProto {
- optional SegmentStateProto segmentState = 1;
- optional uint64 acceptedInEpoch = 2;
- required uint64 lastWriterEpoch = 3;
- // The highest committed txid that this logger has ever seen.
- // This may be higher than the data it actually has, in the case
- // that it was lagging before the old writer crashed.
- optional uint64 lastCommittedTxId = 4;
- }
- /**
- * acceptRecovery()
- */
- message AcceptRecoveryRequestProto {
- required RequestInfoProto reqInfo = 1;
- /** Details on the segment to recover */
- required SegmentStateProto stateToAccept = 2;
-
- /** The URL from which the log may be copied */
- required string fromURL = 3;
- }
- message AcceptRecoveryResponseProto {
- }
- /**
- * Protocol used to journal edits to a JournalNode.
- * See the request and response for details of rpc call.
- */
- service QJournalProtocolService {
- rpc isFormatted(IsFormattedRequestProto) returns (IsFormattedResponseProto);
- rpc getJournalCTime(GetJournalCTimeRequestProto) returns (GetJournalCTimeResponseProto);
-
- rpc doPreUpgrade(DoPreUpgradeRequestProto) returns (DoPreUpgradeResponseProto);
-
- rpc doUpgrade(DoUpgradeRequestProto) returns (DoUpgradeResponseProto);
- rpc doFinalize(DoFinalizeRequestProto) returns (DoFinalizeResponseProto);
- rpc canRollBack(CanRollBackRequestProto) returns (CanRollBackResponseProto);
- rpc doRollback(DoRollbackRequestProto) returns (DoRollbackResponseProto);
- rpc getJournalState(GetJournalStateRequestProto) returns (GetJournalStateResponseProto);
- rpc newEpoch(NewEpochRequestProto) returns (NewEpochResponseProto);
- rpc format(FormatRequestProto) returns (FormatResponseProto);
- rpc journal(JournalRequestProto) returns (JournalResponseProto);
- rpc heartbeat(HeartbeatRequestProto) returns (HeartbeatResponseProto);
- rpc startLogSegment(StartLogSegmentRequestProto)
- returns (StartLogSegmentResponseProto);
- rpc finalizeLogSegment(FinalizeLogSegmentRequestProto)
- returns (FinalizeLogSegmentResponseProto);
- rpc purgeLogs(PurgeLogsRequestProto)
- returns (PurgeLogsResponseProto);
- rpc getEditLogManifest(GetEditLogManifestRequestProto)
- returns (GetEditLogManifestResponseProto);
- rpc prepareRecovery(PrepareRecoveryRequestProto)
- returns (PrepareRecoveryResponseProto);
- rpc acceptRecovery(AcceptRecoveryRequestProto)
- returns (AcceptRecoveryResponseProto);
- }
|