123456789101112131415161718192021222324252627282930313233343536373839404142434445464748495051525354555657585960616263646566676869707172737475767778798081828384858687888990919293949596979899100101102103104105106107108109110111112113114115116117118119120121122123124125126127128129130131132133134135136137138139140141142143144145146147148149150151152153154155156157158159160161162163164165166167168169170171172173174175176177178179180181182183184185186187188189190191192193194195196197198199200201202203204205206207208209210211212213214215216217218219220221222223224225226227228229230231232233234235236237238239240241242243244245246247248249250251252253254255256257258259260261262263264265266267268269270271272273274275276277278279280281282283284285286287288289290291292293294295296297298299300301302303304305306307308309310311312313314315316317318319320321322323324325326327328329330331332333334335336337338339340341342343344345346347348349350351352353354355356357358359360361362363364365366367368369370371372373374375376377378379380381382383384385386387388389390391392393394395396397398399400401402403404405406407408409410411412413414415416417418419420421422423424425426427428429430431432433434435436437438439440441442443444445446447448449450451452453454455456457458459460461462463464465466467468469470471472473474475476477478479 |
- /**
- * 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.
- */
- // This file contains protocol buffers that are used throughout HDFS -- i.e.
- // by the client, server, and data transfer protocols.
- option java_package = "org.apache.hadoop.hdfs.protocol.proto";
- option java_outer_classname = "DatanodeProtocolProtos";
- option java_generic_services = true;
- option java_generate_equals_and_hash = true;
- package hadoop.hdfs.datanode;
- import "hdfs.proto";
- import "erasurecoding.proto";
- import "HdfsServer.proto";
- /**
- * Information to identify a datanode to a namenode
- */
- message DatanodeRegistrationProto {
- required DatanodeIDProto datanodeID = 1; // Datanode information
- required StorageInfoProto storageInfo = 2; // Node information
- required ExportedBlockKeysProto keys = 3; // Block keys
- required string softwareVersion = 4; // Software version of the DN, e.g. "2.0.0"
- }
- /**
- * Commands sent from namenode to the datanodes
- */
- message DatanodeCommandProto {
- enum Type {
- BalancerBandwidthCommand = 0;
- BlockCommand = 1;
- BlockRecoveryCommand = 2;
- FinalizeCommand = 3;
- KeyUpdateCommand = 4;
- RegisterCommand = 5;
- UnusedUpgradeCommand = 6;
- NullDatanodeCommand = 7;
- BlockIdCommand = 8;
- BlockECReconstructionCommand = 9;
- }
- required Type cmdType = 1; // Type of the command
- // One of the following command is available when the corresponding
- // cmdType is set
- optional BalancerBandwidthCommandProto balancerCmd = 2;
- optional BlockCommandProto blkCmd = 3;
- optional BlockRecoveryCommandProto recoveryCmd = 4;
- optional FinalizeCommandProto finalizeCmd = 5;
- optional KeyUpdateCommandProto keyUpdateCmd = 6;
- optional RegisterCommandProto registerCmd = 7;
- optional BlockIdCommandProto blkIdCmd = 8;
- optional BlockECReconstructionCommandProto blkECReconstructionCmd = 9;
- }
- /**
- * Command sent from namenode to datanode to set the
- * maximum bandwidth to be used for balancing.
- */
- message BalancerBandwidthCommandProto {
- // Maximum bandwidth to be used by datanode for balancing
- required uint64 bandwidth = 1;
- }
- /**
- * Command to instruct datanodes to perform certain action
- * on the given set of blocks.
- */
- message BlockCommandProto {
- enum Action {
- TRANSFER = 1; // Transfer blocks to another datanode
- INVALIDATE = 2; // Invalidate blocks
- SHUTDOWN = 3; // Shutdown the datanode
- }
- required Action action = 1;
- required string blockPoolId = 2;
- repeated BlockProto blocks = 3;
- repeated DatanodeInfosProto targets = 4;
- repeated StorageUuidsProto targetStorageUuids = 5;
- repeated StorageTypesProto targetStorageTypes = 6;
- }
- /**
- * Command to instruct datanodes to perform certain action
- * on the given set of block IDs.
- */
- message BlockIdCommandProto {
- enum Action {
- CACHE = 1;
- UNCACHE = 2;
- }
- required Action action = 1;
- required string blockPoolId = 2;
- repeated uint64 blockIds = 3 [packed=true];
- }
- /**
- * List of blocks to be recovered by the datanode
- */
- message BlockRecoveryCommandProto {
- repeated RecoveringBlockProto blocks = 1;
- }
- /**
- * Finalize the upgrade at the datanode
- */
- message FinalizeCommandProto {
- required string blockPoolId = 1; // Block pool to be finalized
- }
- /**
- * Update the block keys at the datanode
- */
- message KeyUpdateCommandProto {
- required ExportedBlockKeysProto keys = 1;
- }
- /**
- * Instruct datanode to register with the namenode
- */
- message RegisterCommandProto {
- // void
- }
- /**
- * Block Erasure coding reconstruction command
- */
- message BlockECReconstructionCommandProto {
- repeated BlockECReconstructionInfoProto blockECReconstructioninfo = 1;
- }
- /**
- * registration - Information of the datanode registering with the namenode
- */
- message RegisterDatanodeRequestProto {
- required DatanodeRegistrationProto registration = 1; // Datanode info
- }
- /**
- * registration - Update registration of the datanode that successfully
- * registered. StorageInfo will be updated to include new
- * storage ID if the datanode did not have one in the request.
- */
- message RegisterDatanodeResponseProto {
- required DatanodeRegistrationProto registration = 1; // Datanode info
- }
- /**
- * failedStorageLocations - storage locations that have failed
- * lastVolumeFailureDate - date/time of last volume failure
- * estimatedCapacityLost - estimate of total capacity lost due to volume failures
- */
- message VolumeFailureSummaryProto {
- repeated string failedStorageLocations = 1;
- required uint64 lastVolumeFailureDate = 2;
- required uint64 estimatedCapacityLostTotal = 3;
- }
- /**
- * registration - datanode registration information
- * capacity - total storage capacity available at the datanode
- * dfsUsed - storage used by HDFS
- * remaining - remaining storage available for HDFS
- * blockPoolUsed - storage used by the block pool
- * xmitsInProgress - number of transfers from this datanode to others
- * xceiverCount - number of active transceiver threads
- * failedVolumes - number of failed volumes. This is redundant with the
- * information included in volumeFailureSummary, but the field is retained
- * for backwards compatibility.
- * cacheCapacity - total cache capacity available at the datanode
- * cacheUsed - amount of cache used
- * volumeFailureSummary - info about volume failures
- * slowPeers - info about peer DataNodes that are suspected to be slow.
- * slowDisks - info about DataNode disks that are suspected to be slow.
- * blksMovementResults - status of the scheduled blocks movements
- */
- message HeartbeatRequestProto {
- required DatanodeRegistrationProto registration = 1; // Datanode info
- repeated StorageReportProto reports = 2;
- optional uint32 xmitsInProgress = 3 [ default = 0 ];
- optional uint32 xceiverCount = 4 [ default = 0 ];
- optional uint32 failedVolumes = 5 [ default = 0 ];
- optional uint64 cacheCapacity = 6 [ default = 0 ];
- optional uint64 cacheUsed = 7 [default = 0 ];
- optional VolumeFailureSummaryProto volumeFailureSummary = 8;
- optional bool requestFullBlockReportLease = 9 [ default = false ];
- repeated SlowPeerReportProto slowPeers = 10;
- repeated SlowDiskReportProto slowDisks = 11;
- }
- /**
- * cmds - Commands from namenode to datanode.
- * haStatus - Status (from an HA perspective) of the NN sending this response
- */
- message HeartbeatResponseProto {
- repeated DatanodeCommandProto cmds = 1; // Returned commands can be null
- required NNHAStatusHeartbeatProto haStatus = 2;
- optional RollingUpgradeStatusProto rollingUpgradeStatus = 3;
- optional RollingUpgradeStatusProto rollingUpgradeStatusV2 = 4;
- optional uint64 fullBlockReportLeaseId = 5 [ default = 0 ];
- }
- /**
- * registration - datanode registration information
- * blockPoolID - block pool ID of the reported blocks
- * blocks - each block is represented as multiple longs in the array.
- * first long represents block ID
- * second long represents length
- * third long represents gen stamp
- * fourth long (if under construction) represents replica state
- * context - An optional field containing information about the context
- * of this block report.
- */
- message BlockReportRequestProto {
- required DatanodeRegistrationProto registration = 1;
- required string blockPoolId = 2;
- repeated StorageBlockReportProto reports = 3;
- optional BlockReportContextProto context = 4;
- }
- message BlockReportContextProto {
- // The total number of RPCs this block report is broken into.
- required int32 totalRpcs = 1;
- // The index of the current RPC (zero-based)
- required int32 curRpc = 2;
- // The unique 64-bit ID of this block report
- required int64 id = 3;
- // The block report lease ID, or 0 if we are sending without a lease to
- // bypass rate-limiting.
- optional uint64 leaseId = 4 [ default = 0 ];
- // True if the reported blocks are sorted by increasing block IDs
- optional bool sorted = 5 [default = false];
- }
- /**
- * Report of blocks in a storage
- */
- message StorageBlockReportProto {
- required DatanodeStorageProto storage = 1; // Storage
- repeated uint64 blocks = 2 [packed=true];
- optional uint64 numberOfBlocks = 3;
- repeated bytes blocksBuffers = 4;
- }
- /**
- * cmd - Command from namenode to the datanode
- */
- message BlockReportResponseProto {
- optional DatanodeCommandProto cmd = 1;
- }
- /**
- * registration - datanode registration information
- * blockPoolId - block pool ID of the reported blocks
- * blocks - representation of blocks as longs for efficiency reasons
- */
- message CacheReportRequestProto {
- required DatanodeRegistrationProto registration = 1;
- required string blockPoolId = 2;
- repeated uint64 blocks = 3 [packed=true];
- }
- message CacheReportResponseProto {
- optional DatanodeCommandProto cmd = 1;
- }
- /**
- * Data structure to send received or deleted block information
- * from datanode to namenode.
- */
- message ReceivedDeletedBlockInfoProto {
- enum BlockStatus {
- RECEIVING = 1; // block being created
- RECEIVED = 2; // block creation complete
- DELETED = 3;
- }
- required BlockProto block = 1;
- required BlockStatus status = 3;
- optional string deleteHint = 2;
- }
- /**
- * List of blocks received and deleted for a storage.
- */
- message StorageReceivedDeletedBlocksProto {
- required string storageUuid = 1 [ deprecated = true ];
- repeated ReceivedDeletedBlockInfoProto blocks = 2;
- optional DatanodeStorageProto storage = 3; // supersedes storageUuid.
- }
- /**
- * registration - datanode registration information
- * blockPoolID - block pool ID of the reported blocks
- * blocks - Received/deleted block list
- */
- message BlockReceivedAndDeletedRequestProto {
- required DatanodeRegistrationProto registration = 1;
- required string blockPoolId = 2;
- repeated StorageReceivedDeletedBlocksProto blocks = 3;
- }
- /**
- * void response
- */
- message BlockReceivedAndDeletedResponseProto {
- }
- /**
- * registartion - Datanode reporting the error
- * errorCode - error code indicating the error
- * msg - Free text description of the error
- */
- message ErrorReportRequestProto {
- enum ErrorCode {
- NOTIFY = 0; // Error report to be logged at the namenode
- DISK_ERROR = 1; // DN has disk errors but still has valid volumes
- INVALID_BLOCK = 2; // Command from namenode has invalid block ID
- FATAL_DISK_ERROR = 3; // No valid volumes left on datanode
- }
- required DatanodeRegistrationProto registartion = 1; // Registartion info
- required uint32 errorCode = 2; // Error code
- required string msg = 3; // Error message
- }
- /**
- * void response
- */
- message ErrorReportResponseProto {
- }
- /**
- * blocks - list of blocks that are reported as corrupt
- */
- message ReportBadBlocksRequestProto {
- repeated LocatedBlockProto blocks = 1;
- }
- /**
- * void response
- */
- message ReportBadBlocksResponseProto {
- }
- /**
- * Commit block synchronization request during lease recovery
- */
- message CommitBlockSynchronizationRequestProto {
- required ExtendedBlockProto block = 1;
- required uint64 newGenStamp = 2;
- required uint64 newLength = 3;
- required bool closeFile = 4;
- required bool deleteBlock = 5;
- repeated DatanodeIDProto newTaragets = 6;
- repeated string newTargetStorages = 7;
- }
- /**
- * void response
- */
- message CommitBlockSynchronizationResponseProto {
- }
- /**
- * Information about a single slow peer that may be reported by
- * the DataNode to the NameNode as part of the heartbeat request.
- * The message includes the peer's DataNodeId and its
- * aggregate packet latency as observed by the reporting DataNode.
- * (DataNodeId must be transmitted as a string for protocol compability
- * with earlier versions of Hadoop).
- *
- * The exact choice of the aggregate is opaque to the NameNode but it
- * _should_ be chosen consistenly by all DataNodes in the cluster.
- * Examples of aggregates are 90th percentile (good) and mean (not so
- * good).
- */
- message SlowPeerReportProto {
- optional string dataNodeId = 1;
- optional double aggregateLatency = 2;
- }
- /**
- * Information about a single slow disk that may be reported by
- * the DataNode to the NameNode as part of the heartbeat request.
- * The message includes the disk's basePath, mean metadata op latency,
- * mean read io latency and mean write io latency as observed by the DataNode.
- */
- message SlowDiskReportProto {
- optional string basePath = 1;
- optional double meanMetadataOpLatency = 2;
- optional double meanReadIoLatency = 3;
- optional double meanWriteIoLatency = 4;
- }
- /**
- * Protocol used from datanode to the namenode
- * See the request and response for details of rpc call.
- */
- service DatanodeProtocolService {
- /**
- * Register a datanode at a namenode
- */
- rpc registerDatanode(RegisterDatanodeRequestProto)
- returns(RegisterDatanodeResponseProto);
- /**
- * Send heartbeat from datanode to namenode
- */
- rpc sendHeartbeat(HeartbeatRequestProto) returns(HeartbeatResponseProto);
- /**
- * Report blocks at a given datanode to the namenode
- */
- rpc blockReport(BlockReportRequestProto) returns(BlockReportResponseProto);
- /**
- * Report cached blocks at a datanode to the namenode
- */
- rpc cacheReport(CacheReportRequestProto) returns(CacheReportResponseProto);
- /**
- * Incremental block report from the DN. This contains info about recently
- * received and deleted blocks, as well as when blocks start being
- * received.
- */
- rpc blockReceivedAndDeleted(BlockReceivedAndDeletedRequestProto)
- returns(BlockReceivedAndDeletedResponseProto);
- /**
- * Report from a datanode of an error to the active namenode.
- * Used for debugging.
- */
- rpc errorReport(ErrorReportRequestProto) returns(ErrorReportResponseProto);
-
- /**
- * Request the version
- */
- rpc versionRequest(VersionRequestProto) returns(VersionResponseProto);
- /**
- * Report corrupt blocks at the specified location
- */
- rpc reportBadBlocks(ReportBadBlocksRequestProto) returns(ReportBadBlocksResponseProto);
- /**
- * Commit block synchronization during lease recovery.
- */
- rpc commitBlockSynchronization(CommitBlockSynchronizationRequestProto)
- returns(CommitBlockSynchronizationResponseProto);
- }
|