001/**
002 * Licensed to the Apache Software Foundation (ASF) under one
003 * or more contributor license agreements.  See the NOTICE file
004 * distributed with this work for additional information
005 * regarding copyright ownership.  The ASF licenses this file
006 * to you under the Apache License, Version 2.0 (the
007 * "License"); you may not use this file except in compliance
008 * with the License.  You may obtain a copy of the License at
009 *
010 *     http://www.apache.org/licenses/LICENSE-2.0
011 *
012 * Unless required by applicable law or agreed to in writing, software
013 * distributed under the License is distributed on an "AS IS" BASIS,
014 * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
015 * See the License for the specific language governing permissions and
016 * limitations under the License.
017 */
018package org.apache.hadoop.hdfs.protocolPB;
019
020import static com.google.common.base.Preconditions.checkNotNull;
021import static org.apache.hadoop.hdfs.protocol.proto.EncryptionZonesProtos
022    .EncryptionZoneProto;
023import static org.apache.hadoop.hdfs.protocol.proto.HdfsProtos.CipherSuiteProto;
024import static org.apache.hadoop.hdfs.protocol.proto.HdfsProtos.CryptoProtocolVersionProto;
025
026import java.io.EOFException;
027import java.io.IOException;
028import java.io.InputStream;
029import java.util.ArrayList;
030import java.util.Arrays;
031import java.util.EnumSet;
032import java.util.List;
033
034import org.apache.hadoop.fs.CacheFlag;
035import org.apache.hadoop.fs.ContentSummary;
036import org.apache.hadoop.fs.CreateFlag;
037import org.apache.hadoop.fs.FsServerDefaults;
038import org.apache.hadoop.fs.Path;
039import org.apache.hadoop.fs.XAttr;
040import org.apache.hadoop.fs.XAttrSetFlag;
041import org.apache.hadoop.fs.permission.AclEntry;
042import org.apache.hadoop.fs.permission.AclEntryScope;
043import org.apache.hadoop.fs.permission.AclEntryType;
044import org.apache.hadoop.fs.permission.AclStatus;
045import org.apache.hadoop.fs.permission.FsAction;
046import org.apache.hadoop.fs.permission.FsPermission;
047import org.apache.hadoop.ha.HAServiceProtocol.HAServiceState;
048import org.apache.hadoop.ha.proto.HAServiceProtocolProtos;
049import org.apache.hadoop.hdfs.inotify.EventBatch;
050import org.apache.hadoop.hdfs.protocol.BlockStoragePolicy;
051import org.apache.hadoop.hdfs.DFSUtil;
052import org.apache.hadoop.hdfs.StorageType;
053import org.apache.hadoop.hdfs.inotify.Event;
054import org.apache.hadoop.hdfs.inotify.EventBatchList;
055import org.apache.hadoop.hdfs.protocol.Block;
056import org.apache.hadoop.hdfs.protocol.CacheDirectiveEntry;
057import org.apache.hadoop.hdfs.protocol.CacheDirectiveInfo;
058import org.apache.hadoop.hdfs.protocol.CacheDirectiveStats;
059import org.apache.hadoop.hdfs.protocol.CachePoolEntry;
060import org.apache.hadoop.hdfs.protocol.CachePoolInfo;
061import org.apache.hadoop.hdfs.protocol.CachePoolStats;
062import org.apache.hadoop.crypto.CipherOption;
063import org.apache.hadoop.crypto.CipherSuite;
064import org.apache.hadoop.hdfs.protocol.ClientProtocol;
065import org.apache.hadoop.hdfs.protocol.CorruptFileBlocks;
066import org.apache.hadoop.crypto.CryptoProtocolVersion;
067import org.apache.hadoop.hdfs.protocol.DatanodeID;
068import org.apache.hadoop.hdfs.protocol.DatanodeInfo;
069import org.apache.hadoop.hdfs.protocol.DatanodeInfo.AdminStates;
070import org.apache.hadoop.hdfs.protocol.DatanodeLocalInfo;
071import org.apache.hadoop.hdfs.protocol.DirectoryListing;
072import org.apache.hadoop.hdfs.protocol.EncryptionZone;
073import org.apache.hadoop.hdfs.protocol.ExtendedBlock;
074import org.apache.hadoop.fs.FileEncryptionInfo;
075import org.apache.hadoop.hdfs.protocol.FsPermissionExtension;
076import org.apache.hadoop.hdfs.protocol.HdfsConstants.DatanodeReportType;
077import org.apache.hadoop.hdfs.protocol.HdfsConstants.RollingUpgradeAction;
078import org.apache.hadoop.hdfs.protocol.HdfsConstants.SafeModeAction;
079import org.apache.hadoop.hdfs.protocol.HdfsFileStatus;
080import org.apache.hadoop.hdfs.protocol.HdfsLocatedFileStatus;
081import org.apache.hadoop.hdfs.protocol.LocatedBlock;
082import org.apache.hadoop.hdfs.protocol.LocatedBlocks;
083import org.apache.hadoop.hdfs.protocol.RollingUpgradeInfo;
084import org.apache.hadoop.hdfs.protocol.RollingUpgradeStatus;
085import org.apache.hadoop.hdfs.protocol.SnapshotDiffReport;
086import org.apache.hadoop.hdfs.protocol.SnapshotDiffReport.DiffReportEntry;
087import org.apache.hadoop.hdfs.protocol.SnapshotDiffReport.DiffType;
088import org.apache.hadoop.hdfs.protocol.SnapshottableDirectoryStatus;
089import org.apache.hadoop.hdfs.protocol.proto.AclProtos.AclEntryProto;
090import org.apache.hadoop.hdfs.protocol.proto.AclProtos.AclEntryProto.AclEntryScopeProto;
091import org.apache.hadoop.hdfs.protocol.proto.AclProtos.AclEntryProto.AclEntryTypeProto;
092import org.apache.hadoop.hdfs.protocol.proto.AclProtos.AclEntryProto.FsActionProto;
093import org.apache.hadoop.hdfs.protocol.proto.AclProtos.AclStatusProto;
094import org.apache.hadoop.hdfs.protocol.proto.AclProtos.GetAclStatusResponseProto;
095import org.apache.hadoop.hdfs.protocol.proto.ClientNamenodeProtocolProtos;
096import org.apache.hadoop.hdfs.protocol.proto.ClientNamenodeProtocolProtos.CacheDirectiveEntryProto;
097import org.apache.hadoop.hdfs.protocol.proto.ClientNamenodeProtocolProtos.CacheDirectiveInfoExpirationProto;
098import org.apache.hadoop.hdfs.protocol.proto.ClientNamenodeProtocolProtos.CacheDirectiveInfoProto;
099import org.apache.hadoop.hdfs.protocol.proto.ClientNamenodeProtocolProtos.CacheDirectiveStatsProto;
100import org.apache.hadoop.hdfs.protocol.proto.ClientNamenodeProtocolProtos.CacheFlagProto;
101import org.apache.hadoop.hdfs.protocol.proto.ClientNamenodeProtocolProtos.CachePoolEntryProto;
102import org.apache.hadoop.hdfs.protocol.proto.ClientNamenodeProtocolProtos.CachePoolInfoProto;
103import org.apache.hadoop.hdfs.protocol.proto.ClientNamenodeProtocolProtos.CachePoolStatsProto;
104import org.apache.hadoop.hdfs.protocol.proto.ClientNamenodeProtocolProtos.CreateFlagProto;
105import org.apache.hadoop.hdfs.protocol.proto.ClientNamenodeProtocolProtos.DatanodeReportTypeProto;
106import org.apache.hadoop.hdfs.protocol.proto.ClientNamenodeProtocolProtos.DatanodeStorageReportProto;
107import org.apache.hadoop.hdfs.protocol.proto.ClientNamenodeProtocolProtos.GetEditsFromTxidResponseProto;
108import org.apache.hadoop.hdfs.protocol.proto.ClientNamenodeProtocolProtos.GetFsStatsResponseProto;
109import org.apache.hadoop.hdfs.protocol.proto.ClientNamenodeProtocolProtos.RollingUpgradeActionProto;
110import org.apache.hadoop.hdfs.protocol.proto.ClientNamenodeProtocolProtos.RollingUpgradeInfoProto;
111import org.apache.hadoop.hdfs.protocol.proto.ClientNamenodeProtocolProtos.SafeModeActionProto;
112import org.apache.hadoop.hdfs.protocol.proto.DataTransferProtos.ShortCircuitShmIdProto;
113import org.apache.hadoop.hdfs.protocol.proto.DataTransferProtos.ShortCircuitShmSlotProto;
114import org.apache.hadoop.hdfs.protocol.proto.DatanodeProtocolProtos;
115import org.apache.hadoop.hdfs.protocol.proto.DatanodeProtocolProtos.BalancerBandwidthCommandProto;
116import org.apache.hadoop.hdfs.protocol.proto.DatanodeProtocolProtos.BlockCommandProto;
117import org.apache.hadoop.hdfs.protocol.proto.DatanodeProtocolProtos.BlockIdCommandProto;
118import org.apache.hadoop.hdfs.protocol.proto.DatanodeProtocolProtos.BlockRecoveryCommandProto;
119import org.apache.hadoop.hdfs.protocol.proto.DatanodeProtocolProtos.BlockReportContextProto;
120import org.apache.hadoop.hdfs.protocol.proto.DatanodeProtocolProtos.DatanodeCommandProto;
121import org.apache.hadoop.hdfs.protocol.proto.DatanodeProtocolProtos.DatanodeRegistrationProto;
122import org.apache.hadoop.hdfs.protocol.proto.DatanodeProtocolProtos.FinalizeCommandProto;
123import org.apache.hadoop.hdfs.protocol.proto.DatanodeProtocolProtos.KeyUpdateCommandProto;
124import org.apache.hadoop.hdfs.protocol.proto.DatanodeProtocolProtos.NNHAStatusHeartbeatProto;
125import org.apache.hadoop.hdfs.protocol.proto.DatanodeProtocolProtos.ReceivedDeletedBlockInfoProto;
126import org.apache.hadoop.hdfs.protocol.proto.DatanodeProtocolProtos.RegisterCommandProto;
127import org.apache.hadoop.hdfs.protocol.proto.HdfsProtos;
128import org.apache.hadoop.hdfs.protocol.proto.HdfsProtos.BlockKeyProto;
129import org.apache.hadoop.hdfs.protocol.proto.HdfsProtos.BlockProto;
130import org.apache.hadoop.hdfs.protocol.proto.HdfsProtos.BlockStoragePolicyProto;
131import org.apache.hadoop.hdfs.protocol.proto.HdfsProtos.BlockWithLocationsProto;
132import org.apache.hadoop.hdfs.protocol.proto.HdfsProtos.BlocksWithLocationsProto;
133import org.apache.hadoop.hdfs.protocol.proto.HdfsProtos.CheckpointCommandProto;
134import org.apache.hadoop.hdfs.protocol.proto.HdfsProtos.CheckpointSignatureProto;
135import org.apache.hadoop.hdfs.protocol.proto.HdfsProtos.CipherOptionProto;
136import org.apache.hadoop.hdfs.protocol.proto.HdfsProtos.ContentSummaryProto;
137import org.apache.hadoop.hdfs.protocol.proto.HdfsProtos.CorruptFileBlocksProto;
138import org.apache.hadoop.hdfs.protocol.proto.HdfsProtos.DataEncryptionKeyProto;
139import org.apache.hadoop.hdfs.protocol.proto.HdfsProtos.DatanodeIDProto;
140import org.apache.hadoop.hdfs.protocol.proto.HdfsProtos.DatanodeInfoProto;
141import org.apache.hadoop.hdfs.protocol.proto.HdfsProtos.DatanodeInfoProto.AdminState;
142import org.apache.hadoop.hdfs.protocol.proto.HdfsProtos.DatanodeInfosProto;
143import org.apache.hadoop.hdfs.protocol.proto.HdfsProtos.DatanodeLocalInfoProto;
144import org.apache.hadoop.hdfs.protocol.proto.HdfsProtos.DatanodeStorageProto;
145import org.apache.hadoop.hdfs.protocol.proto.HdfsProtos.DatanodeStorageProto.StorageState;
146import org.apache.hadoop.hdfs.protocol.proto.HdfsProtos.DirectoryListingProto;
147import org.apache.hadoop.hdfs.protocol.proto.HdfsProtos.ExportedBlockKeysProto;
148import org.apache.hadoop.hdfs.protocol.proto.HdfsProtos.ExtendedBlockProto;
149import org.apache.hadoop.hdfs.protocol.proto.HdfsProtos.FsPermissionProto;
150import org.apache.hadoop.hdfs.protocol.proto.HdfsProtos.FsServerDefaultsProto;
151import org.apache.hadoop.hdfs.protocol.proto.HdfsProtos.HdfsFileStatusProto;
152import org.apache.hadoop.hdfs.protocol.proto.HdfsProtos.HdfsFileStatusProto.FileType;
153import org.apache.hadoop.hdfs.protocol.proto.HdfsProtos.LocatedBlockProto;
154import org.apache.hadoop.hdfs.protocol.proto.HdfsProtos.LocatedBlockProto.Builder;
155import org.apache.hadoop.hdfs.protocol.proto.HdfsProtos.LocatedBlocksProto;
156import org.apache.hadoop.hdfs.protocol.proto.HdfsProtos.NamenodeCommandProto;
157import org.apache.hadoop.hdfs.protocol.proto.HdfsProtos.NamenodeRegistrationProto;
158import org.apache.hadoop.hdfs.protocol.proto.HdfsProtos.NamenodeRegistrationProto.NamenodeRoleProto;
159import org.apache.hadoop.hdfs.protocol.proto.HdfsProtos.NamespaceInfoProto;
160import org.apache.hadoop.hdfs.protocol.proto.HdfsProtos.RecoveringBlockProto;
161import org.apache.hadoop.hdfs.protocol.proto.HdfsProtos.RemoteEditLogManifestProto;
162import org.apache.hadoop.hdfs.protocol.proto.HdfsProtos.RemoteEditLogProto;
163import org.apache.hadoop.hdfs.protocol.proto.HdfsProtos.ReplicaStateProto;
164import org.apache.hadoop.hdfs.protocol.proto.HdfsProtos.RollingUpgradeStatusProto;
165import org.apache.hadoop.hdfs.protocol.proto.HdfsProtos.SnapshotDiffReportEntryProto;
166import org.apache.hadoop.hdfs.protocol.proto.HdfsProtos.SnapshotDiffReportProto;
167import org.apache.hadoop.hdfs.protocol.proto.HdfsProtos.SnapshottableDirectoryListingProto;
168import org.apache.hadoop.hdfs.protocol.proto.HdfsProtos.SnapshottableDirectoryStatusProto;
169import org.apache.hadoop.hdfs.protocol.proto.HdfsProtos.StorageInfoProto;
170import org.apache.hadoop.hdfs.protocol.proto.HdfsProtos.StorageReportProto;
171import org.apache.hadoop.hdfs.protocol.proto.HdfsProtos.StorageTypeProto;
172import org.apache.hadoop.hdfs.protocol.proto.HdfsProtos.StorageTypesProto;
173import org.apache.hadoop.hdfs.protocol.proto.HdfsProtos.StorageUuidsProto;
174import org.apache.hadoop.hdfs.protocol.proto.InotifyProtos;
175import org.apache.hadoop.hdfs.protocol.proto.JournalProtocolProtos.JournalInfoProto;
176import org.apache.hadoop.hdfs.protocol.proto.XAttrProtos.GetXAttrsResponseProto;
177import org.apache.hadoop.hdfs.protocol.proto.XAttrProtos.ListXAttrsResponseProto;
178import org.apache.hadoop.hdfs.protocol.proto.XAttrProtos.XAttrProto;
179import org.apache.hadoop.hdfs.protocol.proto.XAttrProtos.XAttrProto.XAttrNamespaceProto;
180import org.apache.hadoop.hdfs.protocol.proto.XAttrProtos.XAttrSetFlagProto;
181import org.apache.hadoop.hdfs.security.token.block.BlockKey;
182import org.apache.hadoop.hdfs.security.token.block.BlockTokenIdentifier;
183import org.apache.hadoop.hdfs.security.token.block.DataEncryptionKey;
184import org.apache.hadoop.hdfs.security.token.block.ExportedBlockKeys;
185import org.apache.hadoop.hdfs.security.token.delegation.DelegationTokenIdentifier;
186import org.apache.hadoop.hdfs.server.blockmanagement.BlockStoragePolicySuite;
187import org.apache.hadoop.hdfs.server.common.HdfsServerConstants.NamenodeRole;
188import org.apache.hadoop.hdfs.server.common.HdfsServerConstants.NodeType;
189import org.apache.hadoop.hdfs.server.common.HdfsServerConstants.ReplicaState;
190import org.apache.hadoop.hdfs.server.common.Storage;
191import org.apache.hadoop.hdfs.server.common.StorageInfo;
192import org.apache.hadoop.hdfs.server.namenode.CheckpointSignature;
193import org.apache.hadoop.hdfs.server.namenode.INodeId;
194import org.apache.hadoop.hdfs.server.protocol.BalancerBandwidthCommand;
195import org.apache.hadoop.hdfs.server.protocol.BlockCommand;
196import org.apache.hadoop.hdfs.server.protocol.BlockIdCommand;
197import org.apache.hadoop.hdfs.server.protocol.BlockRecoveryCommand;
198import org.apache.hadoop.hdfs.server.protocol.BlockRecoveryCommand.RecoveringBlock;
199import org.apache.hadoop.hdfs.server.protocol.BlockReportContext;
200import org.apache.hadoop.hdfs.server.protocol.BlocksWithLocations;
201import org.apache.hadoop.hdfs.server.protocol.BlocksWithLocations.BlockWithLocations;
202import org.apache.hadoop.hdfs.server.protocol.CheckpointCommand;
203import org.apache.hadoop.hdfs.server.protocol.DatanodeCommand;
204import org.apache.hadoop.hdfs.server.protocol.DatanodeProtocol;
205import org.apache.hadoop.hdfs.server.protocol.DatanodeRegistration;
206import org.apache.hadoop.hdfs.server.protocol.DatanodeStorage;
207import org.apache.hadoop.hdfs.server.protocol.DatanodeStorage.State;
208import org.apache.hadoop.hdfs.server.protocol.DatanodeStorageReport;
209import org.apache.hadoop.hdfs.server.protocol.FinalizeCommand;
210import org.apache.hadoop.hdfs.server.protocol.JournalInfo;
211import org.apache.hadoop.hdfs.server.protocol.KeyUpdateCommand;
212import org.apache.hadoop.hdfs.server.protocol.NNHAStatusHeartbeat;
213import org.apache.hadoop.hdfs.server.protocol.NamenodeCommand;
214import org.apache.hadoop.hdfs.server.protocol.NamenodeRegistration;
215import org.apache.hadoop.hdfs.server.protocol.NamespaceInfo;
216import org.apache.hadoop.hdfs.server.protocol.ReceivedDeletedBlockInfo;
217import org.apache.hadoop.hdfs.server.protocol.ReceivedDeletedBlockInfo.BlockStatus;
218import org.apache.hadoop.hdfs.server.protocol.RegisterCommand;
219import org.apache.hadoop.hdfs.server.protocol.RemoteEditLog;
220import org.apache.hadoop.hdfs.server.protocol.RemoteEditLogManifest;
221import org.apache.hadoop.hdfs.server.protocol.StorageReport;
222import org.apache.hadoop.hdfs.shortcircuit.ShortCircuitShm.ShmId;
223import org.apache.hadoop.hdfs.shortcircuit.ShortCircuitShm.SlotId;
224import org.apache.hadoop.hdfs.util.ExactSizeInputStream;
225import org.apache.hadoop.io.EnumSetWritable;
226import org.apache.hadoop.io.Text;
227import org.apache.hadoop.security.proto.SecurityProtos.TokenProto;
228import org.apache.hadoop.security.token.Token;
229import org.apache.hadoop.util.DataChecksum;
230
231import com.google.common.base.Preconditions;
232import com.google.common.collect.Lists;
233import com.google.common.primitives.Shorts;
234import com.google.protobuf.ByteString;
235import com.google.protobuf.CodedInputStream;
236
237/**
238 * Utilities for converting protobuf classes to and from implementation classes
239 * and other helper utilities to help in dealing with protobuf.
240 * 
241 * Note that when converting from an internal type to protobuf type, the
242 * converter never return null for protobuf type. The check for internal type
243 * being null must be done before calling the convert() method.
244 */
245public class PBHelper {
246  private static final RegisterCommandProto REG_CMD_PROTO = 
247      RegisterCommandProto.newBuilder().build();
248  private static final RegisterCommand REG_CMD = new RegisterCommand();
249
250  private static final AclEntryScope[] ACL_ENTRY_SCOPE_VALUES =
251      AclEntryScope.values();
252  private static final AclEntryType[] ACL_ENTRY_TYPE_VALUES =
253      AclEntryType.values();
254  private static final FsAction[] FSACTION_VALUES =
255      FsAction.values();
256  private static final XAttr.NameSpace[] XATTR_NAMESPACE_VALUES = 
257      XAttr.NameSpace.values();
258
259  private PBHelper() {
260    /** Hidden constructor */
261  }
262
263  public static ByteString getByteString(byte[] bytes) {
264    return ByteString.copyFrom(bytes);
265  }
266
267  private static <T extends Enum<T>, U extends Enum<U>> U castEnum(T from, U[] to) {
268    return to[from.ordinal()];
269  }
270
271  public static NamenodeRole convert(NamenodeRoleProto role) {
272    switch (role) {
273    case NAMENODE:
274      return NamenodeRole.NAMENODE;
275    case BACKUP:
276      return NamenodeRole.BACKUP;
277    case CHECKPOINT:
278      return NamenodeRole.CHECKPOINT;
279    }
280    return null;
281  }
282
283  public static NamenodeRoleProto convert(NamenodeRole role) {
284    switch (role) {
285    case NAMENODE:
286      return NamenodeRoleProto.NAMENODE;
287    case BACKUP:
288      return NamenodeRoleProto.BACKUP;
289    case CHECKPOINT:
290      return NamenodeRoleProto.CHECKPOINT;
291    }
292    return null;
293  }
294
295  public static BlockStoragePolicy[] convertStoragePolicies(
296      List<BlockStoragePolicyProto> policyProtos) {
297    if (policyProtos == null || policyProtos.size() == 0) {
298      return new BlockStoragePolicy[0];
299    }
300    BlockStoragePolicy[] policies = new BlockStoragePolicy[policyProtos.size()];
301    int i = 0;
302    for (BlockStoragePolicyProto proto : policyProtos) {
303      policies[i++] = convert(proto);
304    }
305    return policies;
306  }
307
308  public static BlockStoragePolicy convert(BlockStoragePolicyProto proto) {
309    List<StorageTypeProto> cList = proto.getCreationPolicy()
310        .getStorageTypesList();
311    StorageType[] creationTypes = convertStorageTypes(cList, cList.size());
312    List<StorageTypeProto> cfList = proto.hasCreationFallbackPolicy() ? proto
313        .getCreationFallbackPolicy().getStorageTypesList() : null;
314    StorageType[] creationFallbackTypes = cfList == null ? StorageType
315        .EMPTY_ARRAY : convertStorageTypes(cfList, cfList.size());
316    List<StorageTypeProto> rfList = proto.hasReplicationFallbackPolicy() ?
317        proto.getReplicationFallbackPolicy().getStorageTypesList() : null;
318    StorageType[] replicationFallbackTypes = rfList == null ? StorageType
319        .EMPTY_ARRAY : convertStorageTypes(rfList, rfList.size());
320    return new BlockStoragePolicy((byte) proto.getPolicyId(), proto.getName(),
321        creationTypes, creationFallbackTypes, replicationFallbackTypes);
322  }
323
324  public static BlockStoragePolicyProto convert(BlockStoragePolicy policy) {
325    BlockStoragePolicyProto.Builder builder = BlockStoragePolicyProto
326        .newBuilder().setPolicyId(policy.getId()).setName(policy.getName());
327    // creation storage types
328    StorageTypesProto creationProto = convert(policy.getStorageTypes());
329    Preconditions.checkArgument(creationProto != null);
330    builder.setCreationPolicy(creationProto);
331    // creation fallback
332    StorageTypesProto creationFallbackProto = convert(
333        policy.getCreationFallbacks());
334    if (creationFallbackProto != null) {
335      builder.setCreationFallbackPolicy(creationFallbackProto);
336    }
337    // replication fallback
338    StorageTypesProto replicationFallbackProto = convert(
339        policy.getReplicationFallbacks());
340    if (replicationFallbackProto != null) {
341      builder.setReplicationFallbackPolicy(replicationFallbackProto);
342    }
343    return builder.build();
344  }
345
346  public static StorageTypesProto convert(StorageType[] types) {
347    if (types == null || types.length == 0) {
348      return null;
349    }
350    List<StorageTypeProto> list = convertStorageTypes(types);
351    return StorageTypesProto.newBuilder().addAllStorageTypes(list).build();
352  }
353
354  public static StorageInfoProto convert(StorageInfo info) {
355    return StorageInfoProto.newBuilder().setClusterID(info.getClusterID())
356        .setCTime(info.getCTime()).setLayoutVersion(info.getLayoutVersion())
357        .setNamespceID(info.getNamespaceID()).build();
358  }
359
360  public static StorageInfo convert(StorageInfoProto info, NodeType type) {
361    return new StorageInfo(info.getLayoutVersion(), info.getNamespceID(),
362        info.getClusterID(), info.getCTime(), type);
363  }
364
365  public static NamenodeRegistrationProto convert(NamenodeRegistration reg) {
366    return NamenodeRegistrationProto.newBuilder()
367        .setHttpAddress(reg.getHttpAddress()).setRole(convert(reg.getRole()))
368        .setRpcAddress(reg.getAddress())
369        .setStorageInfo(convert((StorageInfo) reg)).build();
370  }
371
372  public static NamenodeRegistration convert(NamenodeRegistrationProto reg) {
373    StorageInfo si = convert(reg.getStorageInfo(), NodeType.NAME_NODE);
374    return new NamenodeRegistration(reg.getRpcAddress(), reg.getHttpAddress(),
375        si, convert(reg.getRole()));
376  }
377
378  // DatanodeId
379  public static DatanodeID convert(DatanodeIDProto dn) {
380    return new DatanodeID(dn.getIpAddr(), dn.getHostName(), dn.getDatanodeUuid(),
381        dn.getXferPort(), dn.getInfoPort(), dn.hasInfoSecurePort() ? dn
382        .getInfoSecurePort() : 0, dn.getIpcPort());
383  }
384
385  public static DatanodeIDProto convert(DatanodeID dn) {
386    // For wire compatibility with older versions we transmit the StorageID
387    // which is the same as the DatanodeUuid. Since StorageID is a required
388    // field we pass the empty string if the DatanodeUuid is not yet known.
389    return DatanodeIDProto.newBuilder()
390        .setIpAddr(dn.getIpAddr())
391        .setHostName(dn.getHostName())
392        .setXferPort(dn.getXferPort())
393        .setDatanodeUuid(dn.getDatanodeUuid() != null ? dn.getDatanodeUuid() : "")
394        .setInfoPort(dn.getInfoPort())
395        .setInfoSecurePort(dn.getInfoSecurePort())
396        .setIpcPort(dn.getIpcPort()).build();
397  }
398
399  // Arrays of DatanodeId
400  public static DatanodeIDProto[] convert(DatanodeID[] did) {
401    if (did == null)
402      return null;
403    final int len = did.length;
404    DatanodeIDProto[] result = new DatanodeIDProto[len];
405    for (int i = 0; i < len; ++i) {
406      result[i] = convert(did[i]);
407    }
408    return result;
409  }
410  
411  public static DatanodeID[] convert(DatanodeIDProto[] did) {
412    if (did == null) return null;
413    final int len = did.length;
414    DatanodeID[] result = new DatanodeID[len];
415    for (int i = 0; i < len; ++i) {
416      result[i] = convert(did[i]);
417    }
418    return result;
419  }
420  
421  // Block
422  public static BlockProto convert(Block b) {
423    return BlockProto.newBuilder().setBlockId(b.getBlockId())
424        .setGenStamp(b.getGenerationStamp()).setNumBytes(b.getNumBytes())
425        .build();
426  }
427
428  public static Block convert(BlockProto b) {
429    return new Block(b.getBlockId(), b.getNumBytes(), b.getGenStamp());
430  }
431
432  public static BlockWithLocationsProto convert(BlockWithLocations blk) {
433    return BlockWithLocationsProto.newBuilder()
434        .setBlock(convert(blk.getBlock()))
435        .addAllDatanodeUuids(Arrays.asList(blk.getDatanodeUuids()))
436        .addAllStorageUuids(Arrays.asList(blk.getStorageIDs()))
437        .addAllStorageTypes(convertStorageTypes(blk.getStorageTypes()))
438        .build();
439  }
440
441  public static BlockWithLocations convert(BlockWithLocationsProto b) {
442    final List<String> datanodeUuids = b.getDatanodeUuidsList();
443    final List<String> storageUuids = b.getStorageUuidsList();
444    final List<StorageTypeProto> storageTypes = b.getStorageTypesList();
445    return new BlockWithLocations(convert(b.getBlock()),
446        datanodeUuids.toArray(new String[datanodeUuids.size()]),
447        storageUuids.toArray(new String[storageUuids.size()]),
448        convertStorageTypes(storageTypes, storageUuids.size()));
449  }
450
451  public static BlocksWithLocationsProto convert(BlocksWithLocations blks) {
452    BlocksWithLocationsProto.Builder builder = BlocksWithLocationsProto
453        .newBuilder();
454    for (BlockWithLocations b : blks.getBlocks()) {
455      builder.addBlocks(convert(b));
456    }
457    return builder.build();
458  }
459
460  public static BlocksWithLocations convert(BlocksWithLocationsProto blocks) {
461    List<BlockWithLocationsProto> b = blocks.getBlocksList();
462    BlockWithLocations[] ret = new BlockWithLocations[b.size()];
463    int i = 0;
464    for (BlockWithLocationsProto entry : b) {
465      ret[i++] = convert(entry);
466    }
467    return new BlocksWithLocations(ret);
468  }
469
470  public static BlockKeyProto convert(BlockKey key) {
471    byte[] encodedKey = key.getEncodedKey();
472    ByteString keyBytes = ByteString.copyFrom(encodedKey == null ? 
473        DFSUtil.EMPTY_BYTES : encodedKey);
474    return BlockKeyProto.newBuilder().setKeyId(key.getKeyId())
475        .setKeyBytes(keyBytes).setExpiryDate(key.getExpiryDate()).build();
476  }
477
478  public static BlockKey convert(BlockKeyProto k) {
479    return new BlockKey(k.getKeyId(), k.getExpiryDate(), k.getKeyBytes()
480        .toByteArray());
481  }
482
483  public static ExportedBlockKeysProto convert(ExportedBlockKeys keys) {
484    ExportedBlockKeysProto.Builder builder = ExportedBlockKeysProto
485        .newBuilder();
486    builder.setIsBlockTokenEnabled(keys.isBlockTokenEnabled())
487        .setKeyUpdateInterval(keys.getKeyUpdateInterval())
488        .setTokenLifeTime(keys.getTokenLifetime())
489        .setCurrentKey(convert(keys.getCurrentKey()));
490    for (BlockKey k : keys.getAllKeys()) {
491      builder.addAllKeys(convert(k));
492    }
493    return builder.build();
494  }
495
496  public static ExportedBlockKeys convert(ExportedBlockKeysProto keys) {
497    return new ExportedBlockKeys(keys.getIsBlockTokenEnabled(),
498        keys.getKeyUpdateInterval(), keys.getTokenLifeTime(),
499        convert(keys.getCurrentKey()), convertBlockKeys(keys.getAllKeysList()));
500  }
501
502  public static CheckpointSignatureProto convert(CheckpointSignature s) {
503    return CheckpointSignatureProto.newBuilder()
504        .setBlockPoolId(s.getBlockpoolID())
505        .setCurSegmentTxId(s.getCurSegmentTxId())
506        .setMostRecentCheckpointTxId(s.getMostRecentCheckpointTxId())
507        .setStorageInfo(PBHelper.convert((StorageInfo) s)).build();
508  }
509
510  public static CheckpointSignature convert(CheckpointSignatureProto s) {
511    StorageInfo si = PBHelper.convert(s.getStorageInfo(), NodeType.NAME_NODE);
512    return new CheckpointSignature(si, s.getBlockPoolId(),
513        s.getMostRecentCheckpointTxId(), s.getCurSegmentTxId());
514  }
515
516  public static RemoteEditLogProto convert(RemoteEditLog log) {
517    return RemoteEditLogProto.newBuilder()
518        .setStartTxId(log.getStartTxId())
519        .setEndTxId(log.getEndTxId())
520        .setIsInProgress(log.isInProgress()).build();
521  }
522
523  public static RemoteEditLog convert(RemoteEditLogProto l) {
524    return new RemoteEditLog(l.getStartTxId(), l.getEndTxId(),
525        l.getIsInProgress());
526  }
527
528  public static RemoteEditLogManifestProto convert(
529      RemoteEditLogManifest manifest) {
530    RemoteEditLogManifestProto.Builder builder = RemoteEditLogManifestProto
531        .newBuilder();
532    for (RemoteEditLog log : manifest.getLogs()) {
533      builder.addLogs(convert(log));
534    }
535    return builder.build();
536  }
537
538  public static RemoteEditLogManifest convert(
539      RemoteEditLogManifestProto manifest) {
540    List<RemoteEditLog> logs = new ArrayList<RemoteEditLog>(manifest
541        .getLogsList().size());
542    for (RemoteEditLogProto l : manifest.getLogsList()) {
543      logs.add(convert(l));
544    }
545    return new RemoteEditLogManifest(logs);
546  }
547
548  public static CheckpointCommandProto convert(CheckpointCommand cmd) {
549    return CheckpointCommandProto.newBuilder()
550        .setSignature(convert(cmd.getSignature()))
551        .setNeedToReturnImage(cmd.needToReturnImage()).build();
552  }
553
554  public static NamenodeCommandProto convert(NamenodeCommand cmd) {
555    if (cmd instanceof CheckpointCommand) {
556      return NamenodeCommandProto.newBuilder().setAction(cmd.getAction())
557          .setType(NamenodeCommandProto.Type.CheckPointCommand)
558          .setCheckpointCmd(convert((CheckpointCommand) cmd)).build();
559    }
560    return NamenodeCommandProto.newBuilder()
561        .setType(NamenodeCommandProto.Type.NamenodeCommand)
562        .setAction(cmd.getAction()).build();
563  }
564
565  public static BlockKey[] convertBlockKeys(List<BlockKeyProto> list) {
566    BlockKey[] ret = new BlockKey[list.size()];
567    int i = 0;
568    for (BlockKeyProto k : list) {
569      ret[i++] = convert(k);
570    }
571    return ret;
572  }
573
574  public static NamespaceInfo convert(NamespaceInfoProto info) {
575    StorageInfoProto storage = info.getStorageInfo();
576    return new NamespaceInfo(storage.getNamespceID(), storage.getClusterID(),
577        info.getBlockPoolID(), storage.getCTime(), info.getBuildVersion(),
578        info.getSoftwareVersion());
579  }
580
581  public static NamenodeCommand convert(NamenodeCommandProto cmd) {
582    if (cmd == null) return null;
583    switch (cmd.getType()) {
584    case CheckPointCommand:
585      CheckpointCommandProto chkPt = cmd.getCheckpointCmd();
586      return new CheckpointCommand(PBHelper.convert(chkPt.getSignature()),
587          chkPt.getNeedToReturnImage());
588    default:
589      return new NamenodeCommand(cmd.getAction());
590    }
591  }
592  
593  public static ExtendedBlock convert(ExtendedBlockProto eb) {
594    if (eb == null) return null;
595    return new ExtendedBlock( eb.getPoolId(),  eb.getBlockId(),   eb.getNumBytes(),
596       eb.getGenerationStamp());
597  }
598  
599  public static ExtendedBlockProto convert(final ExtendedBlock b) {
600    if (b == null) return null;
601   return ExtendedBlockProto.newBuilder().
602      setPoolId(b.getBlockPoolId()).
603      setBlockId(b.getBlockId()).
604      setNumBytes(b.getNumBytes()).
605      setGenerationStamp(b.getGenerationStamp()).
606      build();
607  }
608  
609  public static RecoveringBlockProto convert(RecoveringBlock b) {
610    if (b == null) {
611      return null;
612    }
613    LocatedBlockProto lb = PBHelper.convert((LocatedBlock)b);
614    return RecoveringBlockProto.newBuilder().setBlock(lb)
615        .setNewGenStamp(b.getNewGenerationStamp()).build();
616  }
617
618  public static RecoveringBlock convert(RecoveringBlockProto b) {
619    ExtendedBlock block = convert(b.getBlock().getB());
620    DatanodeInfo[] locs = convert(b.getBlock().getLocsList());
621    return new RecoveringBlock(block, locs, b.getNewGenStamp());
622  }
623  
624  public static DatanodeInfoProto.AdminState convert(
625      final DatanodeInfo.AdminStates inAs) {
626    switch (inAs) {
627    case NORMAL: return  DatanodeInfoProto.AdminState.NORMAL;
628    case DECOMMISSION_INPROGRESS: 
629        return DatanodeInfoProto.AdminState.DECOMMISSION_INPROGRESS;
630    case DECOMMISSIONED: return DatanodeInfoProto.AdminState.DECOMMISSIONED;
631    default: return DatanodeInfoProto.AdminState.NORMAL;
632    }
633  }
634  
635  static public DatanodeInfo convert(DatanodeInfoProto di) {
636    if (di == null) return null;
637    return new DatanodeInfo(
638        PBHelper.convert(di.getId()),
639        di.hasLocation() ? di.getLocation() : null , 
640        di.getCapacity(),  di.getDfsUsed(),  di.getRemaining(),
641        di.getBlockPoolUsed(), di.getCacheCapacity(), di.getCacheUsed(),
642        di.getLastUpdate(), di.getXceiverCount(),
643        PBHelper.convert(di.getAdminState())); 
644  }
645  
646  static public DatanodeInfoProto convertDatanodeInfo(DatanodeInfo di) {
647    if (di == null) return null;
648    return convert(di);
649  }
650  
651  
652  static public DatanodeInfo[] convert(DatanodeInfoProto di[]) {
653    if (di == null) return null;
654    DatanodeInfo[] result = new DatanodeInfo[di.length];
655    for (int i = 0; i < di.length; i++) {
656      result[i] = convert(di[i]);
657    }    
658    return result;
659  }
660
661  public static List<? extends HdfsProtos.DatanodeInfoProto> convert(
662      DatanodeInfo[] dnInfos) {
663    return convert(dnInfos, 0);
664  }
665  
666  /**
667   * Copy from {@code dnInfos} to a target of list of same size starting at
668   * {@code startIdx}.
669   */
670  public static List<? extends HdfsProtos.DatanodeInfoProto> convert(
671      DatanodeInfo[] dnInfos, int startIdx) {
672    if (dnInfos == null)
673      return null;
674    ArrayList<HdfsProtos.DatanodeInfoProto> protos = Lists
675        .newArrayListWithCapacity(dnInfos.length);
676    for (int i = startIdx; i < dnInfos.length; i++) {
677      protos.add(convert(dnInfos[i]));
678    }
679    return protos;
680  }
681
682  public static DatanodeInfo[] convert(List<DatanodeInfoProto> list) {
683    DatanodeInfo[] info = new DatanodeInfo[list.size()];
684    for (int i = 0; i < info.length; i++) {
685      info[i] = convert(list.get(i));
686    }
687    return info;
688  }
689  
690  public static DatanodeInfoProto convert(DatanodeInfo info) {
691    DatanodeInfoProto.Builder builder = DatanodeInfoProto.newBuilder();
692    if (info.getNetworkLocation() != null) {
693      builder.setLocation(info.getNetworkLocation());
694    }
695    builder
696        .setId(PBHelper.convert((DatanodeID)info))
697        .setCapacity(info.getCapacity())
698        .setDfsUsed(info.getDfsUsed())
699        .setRemaining(info.getRemaining())
700        .setBlockPoolUsed(info.getBlockPoolUsed())
701        .setCacheCapacity(info.getCacheCapacity())
702        .setCacheUsed(info.getCacheUsed())
703        .setLastUpdate(info.getLastUpdate())
704        .setXceiverCount(info.getXceiverCount())
705        .setAdminState(PBHelper.convert(info.getAdminState()))
706        .build();
707    return builder.build();
708  }
709
710  public static DatanodeStorageReportProto convertDatanodeStorageReport(
711      DatanodeStorageReport report) {
712    return DatanodeStorageReportProto.newBuilder()
713        .setDatanodeInfo(convert(report.getDatanodeInfo()))
714        .addAllStorageReports(convertStorageReports(report.getStorageReports()))
715        .build();
716  }
717
718  public static List<DatanodeStorageReportProto> convertDatanodeStorageReports(
719      DatanodeStorageReport[] reports) {
720    final List<DatanodeStorageReportProto> protos
721        = new ArrayList<DatanodeStorageReportProto>(reports.length);
722    for(int i = 0; i < reports.length; i++) {
723      protos.add(convertDatanodeStorageReport(reports[i]));
724    }
725    return protos;
726  }
727
728  public static DatanodeStorageReport convertDatanodeStorageReport(
729      DatanodeStorageReportProto proto) {
730    return new DatanodeStorageReport(
731        convert(proto.getDatanodeInfo()),
732        convertStorageReports(proto.getStorageReportsList()));
733  }
734
735  public static DatanodeStorageReport[] convertDatanodeStorageReports(
736      List<DatanodeStorageReportProto> protos) {
737    final DatanodeStorageReport[] reports
738        = new DatanodeStorageReport[protos.size()];
739    for(int i = 0; i < reports.length; i++) {
740      reports[i] = convertDatanodeStorageReport(protos.get(i));
741    }
742    return reports;
743  }
744
745  public static AdminStates convert(AdminState adminState) {
746    switch(adminState) {
747    case DECOMMISSION_INPROGRESS:
748      return AdminStates.DECOMMISSION_INPROGRESS;
749    case DECOMMISSIONED:
750      return AdminStates.DECOMMISSIONED;
751    case NORMAL:
752    default:
753      return AdminStates.NORMAL;
754    }
755  }
756  
757  public static LocatedBlockProto convert(LocatedBlock b) {
758    if (b == null) return null;
759    Builder builder = LocatedBlockProto.newBuilder();
760    DatanodeInfo[] locs = b.getLocations();
761    List<DatanodeInfo> cachedLocs =
762        Lists.newLinkedList(Arrays.asList(b.getCachedLocations()));
763    for (int i = 0; i < locs.length; i++) {
764      DatanodeInfo loc = locs[i];
765      builder.addLocs(i, PBHelper.convert(loc));
766      boolean locIsCached = cachedLocs.contains(loc);
767      builder.addIsCached(locIsCached);
768      if (locIsCached) {
769        cachedLocs.remove(loc);
770      }
771    }
772    Preconditions.checkArgument(cachedLocs.size() == 0,
773        "Found additional cached replica locations that are not in the set of"
774        + " storage-backed locations!");
775
776    StorageType[] storageTypes = b.getStorageTypes();
777    if (storageTypes != null) {
778      for (int i = 0; i < storageTypes.length; ++i) {
779        builder.addStorageTypes(PBHelper.convertStorageType(storageTypes[i]));
780      }
781    }
782    final String[] storageIDs = b.getStorageIDs();
783    if (storageIDs != null) {
784      builder.addAllStorageIDs(Arrays.asList(storageIDs));
785    }
786
787    return builder.setB(PBHelper.convert(b.getBlock()))
788        .setBlockToken(PBHelper.convert(b.getBlockToken()))
789        .setCorrupt(b.isCorrupt()).setOffset(b.getStartOffset()).build();
790  }
791  
792  public static LocatedBlock convert(LocatedBlockProto proto) {
793    if (proto == null) return null;
794    List<DatanodeInfoProto> locs = proto.getLocsList();
795    DatanodeInfo[] targets = new DatanodeInfo[locs.size()];
796    for (int i = 0; i < locs.size(); i++) {
797      targets[i] = PBHelper.convert(locs.get(i));
798    }
799
800    final StorageType[] storageTypes = convertStorageTypes(
801        proto.getStorageTypesList(), locs.size());
802
803    final int storageIDsCount = proto.getStorageIDsCount();
804    final String[] storageIDs;
805    if (storageIDsCount == 0) {
806      storageIDs = null;
807    } else {
808      Preconditions.checkState(storageIDsCount == locs.size());
809      storageIDs = proto.getStorageIDsList().toArray(new String[storageIDsCount]);
810    }
811
812    // Set values from the isCached list, re-using references from loc
813    List<DatanodeInfo> cachedLocs = new ArrayList<DatanodeInfo>(locs.size());
814    List<Boolean> isCachedList = proto.getIsCachedList();
815    for (int i=0; i<isCachedList.size(); i++) {
816      if (isCachedList.get(i)) {
817        cachedLocs.add(targets[i]);
818      }
819    }
820
821    LocatedBlock lb = new LocatedBlock(PBHelper.convert(proto.getB()), targets,
822        storageIDs, storageTypes, proto.getOffset(), proto.getCorrupt(),
823        cachedLocs.toArray(new DatanodeInfo[0]));
824    lb.setBlockToken(PBHelper.convert(proto.getBlockToken()));
825
826    return lb;
827  }
828
829  public static TokenProto convert(Token<?> tok) {
830    return TokenProto.newBuilder().
831              setIdentifier(ByteString.copyFrom(tok.getIdentifier())).
832              setPassword(ByteString.copyFrom(tok.getPassword())).
833              setKind(tok.getKind().toString()).
834              setService(tok.getService().toString()).build(); 
835  }
836  
837  public static Token<BlockTokenIdentifier> convert(
838      TokenProto blockToken) {
839    return new Token<BlockTokenIdentifier>(blockToken.getIdentifier()
840        .toByteArray(), blockToken.getPassword().toByteArray(), new Text(
841        blockToken.getKind()), new Text(blockToken.getService()));
842  }
843
844  
845  public static Token<DelegationTokenIdentifier> convertDelegationToken(
846      TokenProto blockToken) {
847    return new Token<DelegationTokenIdentifier>(blockToken.getIdentifier()
848        .toByteArray(), blockToken.getPassword().toByteArray(), new Text(
849        blockToken.getKind()), new Text(blockToken.getService()));
850  }
851
852  public static ReplicaState convert(ReplicaStateProto state) {
853    switch (state) {
854    case RBW:
855      return ReplicaState.RBW;
856    case RUR:
857      return ReplicaState.RUR;
858    case RWR:
859      return ReplicaState.RWR;
860    case TEMPORARY:
861      return ReplicaState.TEMPORARY;
862    case FINALIZED:
863    default:
864      return ReplicaState.FINALIZED;
865    }
866  }
867
868  public static ReplicaStateProto convert(ReplicaState state) {
869    switch (state) {
870    case RBW:
871      return ReplicaStateProto.RBW;
872    case RUR:
873      return ReplicaStateProto.RUR;
874    case RWR:
875      return ReplicaStateProto.RWR;
876    case TEMPORARY:
877      return ReplicaStateProto.TEMPORARY;
878    case FINALIZED:
879    default:
880      return ReplicaStateProto.FINALIZED;
881    }
882  }
883  
884  public static DatanodeRegistrationProto convert(
885      DatanodeRegistration registration) {
886    DatanodeRegistrationProto.Builder builder = DatanodeRegistrationProto
887        .newBuilder();
888    return builder.setDatanodeID(PBHelper.convert((DatanodeID) registration))
889        .setStorageInfo(PBHelper.convert(registration.getStorageInfo()))
890        .setKeys(PBHelper.convert(registration.getExportedKeys()))
891        .setSoftwareVersion(registration.getSoftwareVersion()).build();
892  }
893
894  public static DatanodeRegistration convert(DatanodeRegistrationProto proto) {
895    StorageInfo si = convert(proto.getStorageInfo(), NodeType.DATA_NODE);
896    return new DatanodeRegistration(PBHelper.convert(proto.getDatanodeID()),
897        si, PBHelper.convert(proto.getKeys()), proto.getSoftwareVersion());
898  }
899
900  public static DatanodeCommand convert(DatanodeCommandProto proto) {
901    switch (proto.getCmdType()) {
902    case BalancerBandwidthCommand:
903      return PBHelper.convert(proto.getBalancerCmd());
904    case BlockCommand:
905      return PBHelper.convert(proto.getBlkCmd());
906    case BlockRecoveryCommand:
907      return PBHelper.convert(proto.getRecoveryCmd());
908    case FinalizeCommand:
909      return PBHelper.convert(proto.getFinalizeCmd());
910    case KeyUpdateCommand:
911      return PBHelper.convert(proto.getKeyUpdateCmd());
912    case RegisterCommand:
913      return REG_CMD;
914    case BlockIdCommand:
915      return PBHelper.convert(proto.getBlkIdCmd());
916    default:
917      return null;
918    }
919  }
920  
921  public static BalancerBandwidthCommandProto convert(
922      BalancerBandwidthCommand bbCmd) {
923    return BalancerBandwidthCommandProto.newBuilder()
924        .setBandwidth(bbCmd.getBalancerBandwidthValue()).build();
925  }
926
927  public static KeyUpdateCommandProto convert(KeyUpdateCommand cmd) {
928    return KeyUpdateCommandProto.newBuilder()
929        .setKeys(PBHelper.convert(cmd.getExportedKeys())).build();
930  }
931
932  public static BlockRecoveryCommandProto convert(BlockRecoveryCommand cmd) {
933    BlockRecoveryCommandProto.Builder builder = BlockRecoveryCommandProto
934        .newBuilder();
935    for (RecoveringBlock b : cmd.getRecoveringBlocks()) {
936      builder.addBlocks(PBHelper.convert(b));
937    }
938    return builder.build();
939  }
940
941  public static FinalizeCommandProto convert(FinalizeCommand cmd) {
942    return FinalizeCommandProto.newBuilder()
943        .setBlockPoolId(cmd.getBlockPoolId()).build();
944  }
945
946  public static BlockCommandProto convert(BlockCommand cmd) {
947    BlockCommandProto.Builder builder = BlockCommandProto.newBuilder()
948        .setBlockPoolId(cmd.getBlockPoolId());
949    switch (cmd.getAction()) {
950    case DatanodeProtocol.DNA_TRANSFER:
951      builder.setAction(BlockCommandProto.Action.TRANSFER);
952      break;
953    case DatanodeProtocol.DNA_INVALIDATE:
954      builder.setAction(BlockCommandProto.Action.INVALIDATE);
955      break;
956    case DatanodeProtocol.DNA_SHUTDOWN:
957      builder.setAction(BlockCommandProto.Action.SHUTDOWN);
958      break;
959    default:
960      throw new AssertionError("Invalid action");
961    }
962    Block[] blocks = cmd.getBlocks();
963    for (int i = 0; i < blocks.length; i++) {
964      builder.addBlocks(PBHelper.convert(blocks[i]));
965    }
966    builder.addAllTargets(convert(cmd.getTargets()))
967           .addAllTargetStorageUuids(convert(cmd.getTargetStorageIDs()));
968    StorageType[][] types = cmd.getTargetStorageTypes();
969    if (types != null) {
970      builder.addAllTargetStorageTypes(convert(types));
971    }
972    return builder.build();
973  }
974
975  private static List<StorageTypesProto> convert(StorageType[][] types) {
976    List<StorageTypesProto> list = Lists.newArrayList();
977    if (types != null) {
978      for (StorageType[] ts : types) {
979        StorageTypesProto.Builder builder = StorageTypesProto.newBuilder();
980        builder.addAllStorageTypes(convertStorageTypes(ts));
981        list.add(builder.build());
982      }
983    }
984    return list;
985  }
986
987  public static BlockIdCommandProto convert(BlockIdCommand cmd) {
988    BlockIdCommandProto.Builder builder = BlockIdCommandProto.newBuilder()
989        .setBlockPoolId(cmd.getBlockPoolId());
990    switch (cmd.getAction()) {
991    case DatanodeProtocol.DNA_CACHE:
992      builder.setAction(BlockIdCommandProto.Action.CACHE);
993      break;
994    case DatanodeProtocol.DNA_UNCACHE:
995      builder.setAction(BlockIdCommandProto.Action.UNCACHE);
996      break;
997    default:
998      throw new AssertionError("Invalid action");
999    }
1000    long[] blockIds = cmd.getBlockIds();
1001    for (int i = 0; i < blockIds.length; i++) {
1002      builder.addBlockIds(blockIds[i]);
1003    }
1004    return builder.build();
1005  }
1006
1007  private static List<DatanodeInfosProto> convert(DatanodeInfo[][] targets) {
1008    DatanodeInfosProto[] ret = new DatanodeInfosProto[targets.length];
1009    for (int i = 0; i < targets.length; i++) {
1010      ret[i] = DatanodeInfosProto.newBuilder()
1011          .addAllDatanodes(PBHelper.convert(targets[i])).build();
1012    }
1013    return Arrays.asList(ret);
1014  }
1015
1016  private static List<StorageUuidsProto> convert(String[][] targetStorageUuids) {
1017    StorageUuidsProto[] ret = new StorageUuidsProto[targetStorageUuids.length];
1018    for (int i = 0; i < targetStorageUuids.length; i++) {
1019      ret[i] = StorageUuidsProto.newBuilder()
1020          .addAllStorageUuids(Arrays.asList(targetStorageUuids[i])).build();
1021    }
1022    return Arrays.asList(ret);
1023  }
1024
1025  public static DatanodeCommandProto convert(DatanodeCommand datanodeCommand) {
1026    DatanodeCommandProto.Builder builder = DatanodeCommandProto.newBuilder();
1027    if (datanodeCommand == null) {
1028      return builder.setCmdType(DatanodeCommandProto.Type.NullDatanodeCommand)
1029          .build();
1030    }
1031    switch (datanodeCommand.getAction()) {
1032    case DatanodeProtocol.DNA_BALANCERBANDWIDTHUPDATE:
1033      builder.setCmdType(DatanodeCommandProto.Type.BalancerBandwidthCommand)
1034          .setBalancerCmd(
1035              PBHelper.convert((BalancerBandwidthCommand) datanodeCommand));
1036      break;
1037    case DatanodeProtocol.DNA_ACCESSKEYUPDATE:
1038      builder
1039          .setCmdType(DatanodeCommandProto.Type.KeyUpdateCommand)
1040          .setKeyUpdateCmd(PBHelper.convert((KeyUpdateCommand) datanodeCommand));
1041      break;
1042    case DatanodeProtocol.DNA_RECOVERBLOCK:
1043      builder.setCmdType(DatanodeCommandProto.Type.BlockRecoveryCommand)
1044          .setRecoveryCmd(
1045              PBHelper.convert((BlockRecoveryCommand) datanodeCommand));
1046      break;
1047    case DatanodeProtocol.DNA_FINALIZE:
1048      builder.setCmdType(DatanodeCommandProto.Type.FinalizeCommand)
1049          .setFinalizeCmd(PBHelper.convert((FinalizeCommand) datanodeCommand));
1050      break;
1051    case DatanodeProtocol.DNA_REGISTER:
1052      builder.setCmdType(DatanodeCommandProto.Type.RegisterCommand)
1053          .setRegisterCmd(REG_CMD_PROTO);
1054      break;
1055    case DatanodeProtocol.DNA_TRANSFER:
1056    case DatanodeProtocol.DNA_INVALIDATE:
1057    case DatanodeProtocol.DNA_SHUTDOWN:
1058      builder.setCmdType(DatanodeCommandProto.Type.BlockCommand).
1059        setBlkCmd(PBHelper.convert((BlockCommand) datanodeCommand));
1060      break;
1061    case DatanodeProtocol.DNA_CACHE:
1062    case DatanodeProtocol.DNA_UNCACHE:
1063      builder.setCmdType(DatanodeCommandProto.Type.BlockIdCommand).
1064        setBlkIdCmd(PBHelper.convert((BlockIdCommand) datanodeCommand));
1065      break;
1066    case DatanodeProtocol.DNA_UNKNOWN: //Not expected
1067    default:
1068      builder.setCmdType(DatanodeCommandProto.Type.NullDatanodeCommand);
1069    }
1070    return builder.build();
1071  }
1072
1073  public static KeyUpdateCommand convert(KeyUpdateCommandProto keyUpdateCmd) {
1074    return new KeyUpdateCommand(PBHelper.convert(keyUpdateCmd.getKeys()));
1075  }
1076
1077  public static FinalizeCommand convert(FinalizeCommandProto finalizeCmd) {
1078    return new FinalizeCommand(finalizeCmd.getBlockPoolId());
1079  }
1080
1081  public static BlockRecoveryCommand convert(
1082      BlockRecoveryCommandProto recoveryCmd) {
1083    List<RecoveringBlockProto> list = recoveryCmd.getBlocksList();
1084    List<RecoveringBlock> recoveringBlocks = new ArrayList<RecoveringBlock>(
1085        list.size());
1086    
1087    for (RecoveringBlockProto rbp : list) {
1088      recoveringBlocks.add(PBHelper.convert(rbp));
1089    }
1090    return new BlockRecoveryCommand(recoveringBlocks);
1091  }
1092
1093  public static BlockCommand convert(BlockCommandProto blkCmd) {
1094    List<BlockProto> blockProtoList = blkCmd.getBlocksList();
1095    Block[] blocks = new Block[blockProtoList.size()];
1096    for (int i = 0; i < blockProtoList.size(); i++) {
1097      blocks[i] = PBHelper.convert(blockProtoList.get(i));
1098    }
1099    List<DatanodeInfosProto> targetList = blkCmd.getTargetsList();
1100    DatanodeInfo[][] targets = new DatanodeInfo[targetList.size()][];
1101    for (int i = 0; i < targetList.size(); i++) {
1102      targets[i] = PBHelper.convert(targetList.get(i));
1103    }
1104
1105    StorageType[][] targetStorageTypes = new StorageType[targetList.size()][];
1106    List<StorageTypesProto> targetStorageTypesList = blkCmd.getTargetStorageTypesList();
1107    if (targetStorageTypesList.isEmpty()) { // missing storage types
1108      for(int i = 0; i < targetStorageTypes.length; i++) {
1109        targetStorageTypes[i] = new StorageType[targets[i].length];
1110        Arrays.fill(targetStorageTypes[i], StorageType.DEFAULT);
1111      }
1112    } else {
1113      for(int i = 0; i < targetStorageTypes.length; i++) {
1114        List<StorageTypeProto> p = targetStorageTypesList.get(i).getStorageTypesList();
1115        targetStorageTypes[i] = convertStorageTypes(p, targets[i].length);
1116      }
1117    }
1118
1119    List<StorageUuidsProto> targetStorageUuidsList = blkCmd.getTargetStorageUuidsList();
1120    String[][] targetStorageIDs = new String[targetStorageUuidsList.size()][];
1121    for(int i = 0; i < targetStorageIDs.length; i++) {
1122      List<String> storageIDs = targetStorageUuidsList.get(i).getStorageUuidsList();
1123      targetStorageIDs[i] = storageIDs.toArray(new String[storageIDs.size()]);
1124    }
1125
1126    int action = DatanodeProtocol.DNA_UNKNOWN;
1127    switch (blkCmd.getAction()) {
1128    case TRANSFER:
1129      action = DatanodeProtocol.DNA_TRANSFER;
1130      break;
1131    case INVALIDATE:
1132      action = DatanodeProtocol.DNA_INVALIDATE;
1133      break;
1134    case SHUTDOWN:
1135      action = DatanodeProtocol.DNA_SHUTDOWN;
1136      break;
1137    default:
1138      throw new AssertionError("Unknown action type: " + blkCmd.getAction());
1139    }
1140    return new BlockCommand(action, blkCmd.getBlockPoolId(), blocks, targets,
1141        targetStorageTypes, targetStorageIDs);
1142  }
1143
1144  public static BlockIdCommand convert(BlockIdCommandProto blkIdCmd) {
1145    int numBlockIds = blkIdCmd.getBlockIdsCount();
1146    long blockIds[] = new long[numBlockIds];
1147    for (int i = 0; i < numBlockIds; i++) {
1148      blockIds[i] = blkIdCmd.getBlockIds(i);
1149    }
1150    int action = DatanodeProtocol.DNA_UNKNOWN;
1151    switch (blkIdCmd.getAction()) {
1152    case CACHE:
1153      action = DatanodeProtocol.DNA_CACHE;
1154      break;
1155    case UNCACHE:
1156      action = DatanodeProtocol.DNA_UNCACHE;
1157      break;
1158    default:
1159      throw new AssertionError("Unknown action type: " + blkIdCmd.getAction());
1160    }
1161    return new BlockIdCommand(action, blkIdCmd.getBlockPoolId(), blockIds);
1162  }
1163
1164  public static DatanodeInfo[] convert(DatanodeInfosProto datanodeInfosProto) {
1165    List<DatanodeInfoProto> proto = datanodeInfosProto.getDatanodesList();
1166    DatanodeInfo[] infos = new DatanodeInfo[proto.size()];
1167    for (int i = 0; i < infos.length; i++) {
1168      infos[i] = PBHelper.convert(proto.get(i));
1169    }
1170    return infos;
1171  }
1172
1173  public static BalancerBandwidthCommand convert(
1174      BalancerBandwidthCommandProto balancerCmd) {
1175    return new BalancerBandwidthCommand(balancerCmd.getBandwidth());
1176  }
1177
1178  public static ReceivedDeletedBlockInfoProto convert(
1179      ReceivedDeletedBlockInfo receivedDeletedBlockInfo) {
1180    ReceivedDeletedBlockInfoProto.Builder builder = 
1181        ReceivedDeletedBlockInfoProto.newBuilder();
1182    
1183    ReceivedDeletedBlockInfoProto.BlockStatus status;
1184    switch (receivedDeletedBlockInfo.getStatus()) {
1185    case RECEIVING_BLOCK:
1186      status = ReceivedDeletedBlockInfoProto.BlockStatus.RECEIVING;
1187      break;
1188    case RECEIVED_BLOCK:
1189      status = ReceivedDeletedBlockInfoProto.BlockStatus.RECEIVED;
1190      break;
1191    case DELETED_BLOCK:
1192      status = ReceivedDeletedBlockInfoProto.BlockStatus.DELETED;
1193      break;
1194    default:
1195      throw new IllegalArgumentException("Bad status: " +
1196          receivedDeletedBlockInfo.getStatus());
1197    }
1198    builder.setStatus(status);
1199    
1200    if (receivedDeletedBlockInfo.getDelHints() != null) {
1201      builder.setDeleteHint(receivedDeletedBlockInfo.getDelHints());
1202    }
1203    return builder.setBlock(PBHelper.convert(receivedDeletedBlockInfo.getBlock()))
1204        .build();
1205  }
1206
1207  public static ReceivedDeletedBlockInfo convert(
1208      ReceivedDeletedBlockInfoProto proto) {
1209    ReceivedDeletedBlockInfo.BlockStatus status = null;
1210    switch (proto.getStatus()) {
1211    case RECEIVING:
1212      status = BlockStatus.RECEIVING_BLOCK;
1213      break;
1214    case RECEIVED:
1215      status = BlockStatus.RECEIVED_BLOCK;
1216      break;
1217    case DELETED:
1218      status = BlockStatus.DELETED_BLOCK;
1219      break;
1220    }
1221    return new ReceivedDeletedBlockInfo(
1222        PBHelper.convert(proto.getBlock()),
1223        status,
1224        proto.hasDeleteHint() ? proto.getDeleteHint() : null);
1225  }
1226  
1227  public static NamespaceInfoProto convert(NamespaceInfo info) {
1228    return NamespaceInfoProto.newBuilder()
1229        .setBlockPoolID(info.getBlockPoolID())
1230        .setBuildVersion(info.getBuildVersion())
1231        .setUnused(0)
1232        .setStorageInfo(PBHelper.convert((StorageInfo)info))
1233        .setSoftwareVersion(info.getSoftwareVersion()).build();
1234  }
1235  
1236  // Located Block Arrays and Lists
1237  public static LocatedBlockProto[] convertLocatedBlock(LocatedBlock[] lb) {
1238    if (lb == null) return null;
1239    return convertLocatedBlock2(Arrays.asList(lb)).toArray(
1240        new LocatedBlockProto[lb.length]);
1241  }
1242  
1243  public static LocatedBlock[] convertLocatedBlock(LocatedBlockProto[] lb) {
1244    if (lb == null) return null;
1245    return convertLocatedBlock(Arrays.asList(lb)).toArray(
1246        new LocatedBlock[lb.length]);
1247  }
1248  
1249  public static List<LocatedBlock> convertLocatedBlock(
1250      List<LocatedBlockProto> lb) {
1251    if (lb == null) return null;
1252    final int len = lb.size();
1253    List<LocatedBlock> result = 
1254        new ArrayList<LocatedBlock>(len);
1255    for (int i = 0; i < len; ++i) {
1256      result.add(PBHelper.convert(lb.get(i)));
1257    }
1258    return result;
1259  }
1260  
1261  public static List<LocatedBlockProto> convertLocatedBlock2(List<LocatedBlock> lb) {
1262    if (lb == null) return null;
1263    final int len = lb.size();
1264    List<LocatedBlockProto> result = new ArrayList<LocatedBlockProto>(len);
1265    for (int i = 0; i < len; ++i) {
1266      result.add(PBHelper.convert(lb.get(i)));
1267    }
1268    return result;
1269  }
1270  
1271  
1272  // LocatedBlocks
1273  public static LocatedBlocks convert(LocatedBlocksProto lb) {
1274    return new LocatedBlocks(
1275        lb.getFileLength(), lb.getUnderConstruction(),
1276        PBHelper.convertLocatedBlock(lb.getBlocksList()),
1277        lb.hasLastBlock() ? PBHelper.convert(lb.getLastBlock()) : null,
1278        lb.getIsLastBlockComplete(),
1279        lb.hasFileEncryptionInfo() ? convert(lb.getFileEncryptionInfo()) :
1280            null);
1281  }
1282  
1283  public static LocatedBlocksProto convert(LocatedBlocks lb) {
1284    if (lb == null) {
1285      return null;
1286    }
1287    LocatedBlocksProto.Builder builder = 
1288        LocatedBlocksProto.newBuilder();
1289    if (lb.getLastLocatedBlock() != null) {
1290      builder.setLastBlock(PBHelper.convert(lb.getLastLocatedBlock()));
1291    }
1292    if (lb.getFileEncryptionInfo() != null) {
1293      builder.setFileEncryptionInfo(convert(lb.getFileEncryptionInfo()));
1294    }
1295    return builder.setFileLength(lb.getFileLength())
1296        .setUnderConstruction(lb.isUnderConstruction())
1297        .addAllBlocks(PBHelper.convertLocatedBlock2(lb.getLocatedBlocks()))
1298        .setIsLastBlockComplete(lb.isLastBlockComplete()).build();
1299  }
1300  
1301  // DataEncryptionKey
1302  public static DataEncryptionKey convert(DataEncryptionKeyProto bet) {
1303    String encryptionAlgorithm = bet.getEncryptionAlgorithm();
1304    return new DataEncryptionKey(bet.getKeyId(),
1305        bet.getBlockPoolId(),
1306        bet.getNonce().toByteArray(),
1307        bet.getEncryptionKey().toByteArray(),
1308        bet.getExpiryDate(),
1309        encryptionAlgorithm.isEmpty() ? null : encryptionAlgorithm);
1310  }
1311  
1312  public static DataEncryptionKeyProto convert(DataEncryptionKey bet) {
1313    DataEncryptionKeyProto.Builder b = DataEncryptionKeyProto.newBuilder()
1314        .setKeyId(bet.keyId)
1315        .setBlockPoolId(bet.blockPoolId)
1316        .setNonce(ByteString.copyFrom(bet.nonce))
1317        .setEncryptionKey(ByteString.copyFrom(bet.encryptionKey))
1318        .setExpiryDate(bet.expiryDate);
1319    if (bet.encryptionAlgorithm != null) {
1320      b.setEncryptionAlgorithm(bet.encryptionAlgorithm);
1321    }
1322    return b.build();
1323  }
1324  
1325  public static FsServerDefaults convert(FsServerDefaultsProto fs) {
1326    if (fs == null) return null;
1327    return new FsServerDefaults(
1328        fs.getBlockSize(), fs.getBytesPerChecksum(), 
1329        fs.getWritePacketSize(), (short) fs.getReplication(),
1330        fs.getFileBufferSize(),
1331        fs.getEncryptDataTransfer(),
1332        fs.getTrashInterval(),
1333        PBHelper.convert(fs.getChecksumType()));
1334  }
1335  
1336  public static FsServerDefaultsProto convert(FsServerDefaults fs) {
1337    if (fs == null) return null;
1338    return FsServerDefaultsProto.newBuilder().
1339      setBlockSize(fs.getBlockSize()).
1340      setBytesPerChecksum(fs.getBytesPerChecksum()).
1341      setWritePacketSize(fs.getWritePacketSize())
1342      .setReplication(fs.getReplication())
1343      .setFileBufferSize(fs.getFileBufferSize())
1344      .setEncryptDataTransfer(fs.getEncryptDataTransfer())
1345      .setTrashInterval(fs.getTrashInterval())
1346      .setChecksumType(PBHelper.convert(fs.getChecksumType()))
1347      .build();
1348  }
1349  
1350  public static FsPermissionProto convert(FsPermission p) {
1351    return FsPermissionProto.newBuilder().setPerm(p.toExtendedShort()).build();
1352  }
1353  
1354  public static FsPermission convert(FsPermissionProto p) {
1355    return new FsPermissionExtension((short)p.getPerm());
1356  }
1357  
1358  
1359  // The creatFlag field in PB is a bitmask whose values are the same a the 
1360  // emum values of CreateFlag
1361  public static int convertCreateFlag(EnumSetWritable<CreateFlag> flag) {
1362    int value = 0;
1363    if (flag.contains(CreateFlag.APPEND)) {
1364      value |= CreateFlagProto.APPEND.getNumber();
1365    }
1366    if (flag.contains(CreateFlag.CREATE)) {
1367      value |= CreateFlagProto.CREATE.getNumber();
1368    }
1369    if (flag.contains(CreateFlag.OVERWRITE)) {
1370      value |= CreateFlagProto.OVERWRITE.getNumber();
1371    }
1372    if (flag.contains(CreateFlag.LAZY_PERSIST)) {
1373      value |= CreateFlagProto.LAZY_PERSIST.getNumber();
1374    }
1375    return value;
1376  }
1377  
1378  public static EnumSetWritable<CreateFlag> convertCreateFlag(int flag) {
1379    EnumSet<CreateFlag> result = 
1380       EnumSet.noneOf(CreateFlag.class);   
1381    if ((flag & CreateFlagProto.APPEND_VALUE) == CreateFlagProto.APPEND_VALUE) {
1382      result.add(CreateFlag.APPEND);
1383    }
1384    if ((flag & CreateFlagProto.CREATE_VALUE) == CreateFlagProto.CREATE_VALUE) {
1385      result.add(CreateFlag.CREATE);
1386    }
1387    if ((flag & CreateFlagProto.OVERWRITE_VALUE) 
1388        == CreateFlagProto.OVERWRITE_VALUE) {
1389      result.add(CreateFlag.OVERWRITE);
1390    }
1391    if ((flag & CreateFlagProto.LAZY_PERSIST_VALUE)
1392        == CreateFlagProto.LAZY_PERSIST_VALUE) {
1393      result.add(CreateFlag.LAZY_PERSIST);
1394    }
1395    return new EnumSetWritable<CreateFlag>(result);
1396  }
1397
1398  public static int convertCacheFlags(EnumSet<CacheFlag> flags) {
1399    int value = 0;
1400    if (flags.contains(CacheFlag.FORCE)) {
1401      value |= CacheFlagProto.FORCE.getNumber();
1402    }
1403    return value;
1404  }
1405
1406  public static EnumSet<CacheFlag> convertCacheFlags(int flags) {
1407    EnumSet<CacheFlag> result = EnumSet.noneOf(CacheFlag.class);
1408    if ((flags & CacheFlagProto.FORCE_VALUE) == CacheFlagProto.FORCE_VALUE) {
1409      result.add(CacheFlag.FORCE);
1410    }
1411    return result;
1412  }
1413
1414  public static HdfsFileStatus convert(HdfsFileStatusProto fs) {
1415    if (fs == null)
1416      return null;
1417    return new HdfsLocatedFileStatus(
1418        fs.getLength(), fs.getFileType().equals(FileType.IS_DIR), 
1419        fs.getBlockReplication(), fs.getBlocksize(),
1420        fs.getModificationTime(), fs.getAccessTime(),
1421        PBHelper.convert(fs.getPermission()), fs.getOwner(), fs.getGroup(), 
1422        fs.getFileType().equals(FileType.IS_SYMLINK) ? 
1423            fs.getSymlink().toByteArray() : null,
1424        fs.getPath().toByteArray(),
1425        fs.hasFileId()? fs.getFileId(): INodeId.GRANDFATHER_INODE_ID,
1426        fs.hasLocations() ? PBHelper.convert(fs.getLocations()) : null,
1427        fs.hasChildrenNum() ? fs.getChildrenNum() : -1,
1428        fs.hasFileEncryptionInfo() ? convert(fs.getFileEncryptionInfo()) : null,
1429        fs.hasStoragePolicy() ? (byte) fs.getStoragePolicy()
1430            : BlockStoragePolicySuite.ID_UNSPECIFIED);
1431  }
1432
1433  public static SnapshottableDirectoryStatus convert(
1434      SnapshottableDirectoryStatusProto sdirStatusProto) {
1435    if (sdirStatusProto == null) {
1436      return null;
1437    }
1438    final HdfsFileStatusProto status = sdirStatusProto.getDirStatus();
1439    return new SnapshottableDirectoryStatus(
1440        status.getModificationTime(),
1441        status.getAccessTime(),
1442        PBHelper.convert(status.getPermission()),
1443        status.getOwner(),
1444        status.getGroup(),
1445        status.getPath().toByteArray(),
1446        status.getFileId(),
1447        status.getChildrenNum(),
1448        sdirStatusProto.getSnapshotNumber(),
1449        sdirStatusProto.getSnapshotQuota(),
1450        sdirStatusProto.getParentFullpath().toByteArray());
1451  }
1452  
1453  public static HdfsFileStatusProto convert(HdfsFileStatus fs) {
1454    if (fs == null)
1455      return null;
1456    FileType fType = FileType.IS_FILE;
1457    if (fs.isDir()) {
1458      fType = FileType.IS_DIR;
1459    } else if (fs.isSymlink()) {
1460      fType = FileType.IS_SYMLINK;
1461    }
1462
1463    HdfsFileStatusProto.Builder builder = 
1464     HdfsFileStatusProto.newBuilder().
1465      setLength(fs.getLen()).
1466      setFileType(fType).
1467      setBlockReplication(fs.getReplication()).
1468      setBlocksize(fs.getBlockSize()).
1469      setModificationTime(fs.getModificationTime()).
1470      setAccessTime(fs.getAccessTime()).
1471      setPermission(PBHelper.convert(fs.getPermission())).
1472      setOwner(fs.getOwner()).
1473      setGroup(fs.getGroup()).
1474      setFileId(fs.getFileId()).
1475      setChildrenNum(fs.getChildrenNum()).
1476      setPath(ByteString.copyFrom(fs.getLocalNameInBytes())).
1477      setStoragePolicy(fs.getStoragePolicy());
1478    if (fs.isSymlink())  {
1479      builder.setSymlink(ByteString.copyFrom(fs.getSymlinkInBytes()));
1480    }
1481    if (fs.getFileEncryptionInfo() != null) {
1482      builder.setFileEncryptionInfo(convert(fs.getFileEncryptionInfo()));
1483    }
1484    if (fs instanceof HdfsLocatedFileStatus) {
1485      final HdfsLocatedFileStatus lfs = (HdfsLocatedFileStatus) fs;
1486      LocatedBlocks locations = lfs.getBlockLocations();
1487      if (locations != null) {
1488        builder.setLocations(PBHelper.convert(locations));
1489      }
1490    }
1491    return builder.build();
1492  }
1493  
1494  public static SnapshottableDirectoryStatusProto convert(
1495      SnapshottableDirectoryStatus status) {
1496    if (status == null) {
1497      return null;
1498    }
1499    int snapshotNumber = status.getSnapshotNumber();
1500    int snapshotQuota = status.getSnapshotQuota();
1501    byte[] parentFullPath = status.getParentFullPath();
1502    ByteString parentFullPathBytes = ByteString.copyFrom(
1503        parentFullPath == null ? DFSUtil.EMPTY_BYTES : parentFullPath);
1504    HdfsFileStatusProto fs = convert(status.getDirStatus());
1505    SnapshottableDirectoryStatusProto.Builder builder = 
1506        SnapshottableDirectoryStatusProto
1507        .newBuilder().setSnapshotNumber(snapshotNumber)
1508        .setSnapshotQuota(snapshotQuota).setParentFullpath(parentFullPathBytes)
1509        .setDirStatus(fs);
1510    return builder.build();
1511  }
1512  
1513  public static HdfsFileStatusProto[] convert(HdfsFileStatus[] fs) {
1514    if (fs == null) return null;
1515    final int len = fs.length;
1516    HdfsFileStatusProto[] result = new HdfsFileStatusProto[len];
1517    for (int i = 0; i < len; ++i) {
1518      result[i] = PBHelper.convert(fs[i]);
1519    }
1520    return result;
1521  }
1522  
1523  public static HdfsFileStatus[] convert(HdfsFileStatusProto[] fs) {
1524    if (fs == null) return null;
1525    final int len = fs.length;
1526    HdfsFileStatus[] result = new HdfsFileStatus[len];
1527    for (int i = 0; i < len; ++i) {
1528      result[i] = PBHelper.convert(fs[i]);
1529    }
1530    return result;
1531  }
1532  
1533  public static DirectoryListing convert(DirectoryListingProto dl) {
1534    if (dl == null)
1535      return null;
1536    List<HdfsFileStatusProto> partList =  dl.getPartialListingList();
1537    return new DirectoryListing( 
1538        partList.isEmpty() ? new HdfsLocatedFileStatus[0] 
1539          : PBHelper.convert(
1540              partList.toArray(new HdfsFileStatusProto[partList.size()])),
1541        dl.getRemainingEntries());
1542  }
1543
1544  public static DirectoryListingProto convert(DirectoryListing d) {
1545    if (d == null)
1546      return null;
1547    return DirectoryListingProto.newBuilder().
1548        addAllPartialListing(Arrays.asList(
1549            PBHelper.convert(d.getPartialListing()))).
1550        setRemainingEntries(d.getRemainingEntries()).
1551        build();
1552  }
1553
1554  public static long[] convert(GetFsStatsResponseProto res) {
1555    long[] result = new long[6];
1556    result[ClientProtocol.GET_STATS_CAPACITY_IDX] = res.getCapacity();
1557    result[ClientProtocol.GET_STATS_USED_IDX] = res.getUsed();
1558    result[ClientProtocol.GET_STATS_REMAINING_IDX] = res.getRemaining();
1559    result[ClientProtocol.GET_STATS_UNDER_REPLICATED_IDX] = res.getUnderReplicated();
1560    result[ClientProtocol.GET_STATS_CORRUPT_BLOCKS_IDX] = res.getCorruptBlocks();
1561    result[ClientProtocol.GET_STATS_MISSING_BLOCKS_IDX] = res.getMissingBlocks();
1562    return result;
1563  }
1564  
1565  public static GetFsStatsResponseProto convert(long[] fsStats) {
1566    GetFsStatsResponseProto.Builder result = GetFsStatsResponseProto
1567        .newBuilder();
1568    if (fsStats.length >= ClientProtocol.GET_STATS_CAPACITY_IDX + 1)
1569      result.setCapacity(fsStats[ClientProtocol.GET_STATS_CAPACITY_IDX]);
1570    if (fsStats.length >= ClientProtocol.GET_STATS_USED_IDX + 1)
1571      result.setUsed(fsStats[ClientProtocol.GET_STATS_USED_IDX]);
1572    if (fsStats.length >= ClientProtocol.GET_STATS_REMAINING_IDX + 1)
1573      result.setRemaining(fsStats[ClientProtocol.GET_STATS_REMAINING_IDX]);
1574    if (fsStats.length >= ClientProtocol.GET_STATS_UNDER_REPLICATED_IDX + 1)
1575      result.setUnderReplicated(
1576              fsStats[ClientProtocol.GET_STATS_UNDER_REPLICATED_IDX]);
1577    if (fsStats.length >= ClientProtocol.GET_STATS_CORRUPT_BLOCKS_IDX + 1)
1578      result.setCorruptBlocks(
1579          fsStats[ClientProtocol.GET_STATS_CORRUPT_BLOCKS_IDX]);
1580    if (fsStats.length >= ClientProtocol.GET_STATS_MISSING_BLOCKS_IDX + 1)
1581      result.setMissingBlocks(
1582          fsStats[ClientProtocol.GET_STATS_MISSING_BLOCKS_IDX]);
1583    return result.build();
1584  }
1585  
1586  public static DatanodeReportTypeProto
1587    convert(DatanodeReportType t) {
1588    switch (t) {
1589    case ALL: return DatanodeReportTypeProto.ALL;
1590    case LIVE: return DatanodeReportTypeProto.LIVE;
1591    case DEAD: return DatanodeReportTypeProto.DEAD;
1592    case DECOMMISSIONING: return DatanodeReportTypeProto.DECOMMISSIONING;
1593    default: 
1594      throw new IllegalArgumentException("Unexpected data type report:" + t);
1595    }
1596  }
1597  
1598  public static DatanodeReportType 
1599    convert(DatanodeReportTypeProto t) {
1600    switch (t) {
1601    case ALL: return DatanodeReportType.ALL;
1602    case LIVE: return DatanodeReportType.LIVE;
1603    case DEAD: return DatanodeReportType.DEAD;
1604    case DECOMMISSIONING: return DatanodeReportType.DECOMMISSIONING;
1605    default: 
1606      throw new IllegalArgumentException("Unexpected data type report:" + t);
1607    }
1608  }
1609
1610  public static SafeModeActionProto convert(
1611      SafeModeAction a) {
1612    switch (a) {
1613    case SAFEMODE_LEAVE:
1614      return SafeModeActionProto.SAFEMODE_LEAVE;
1615    case SAFEMODE_ENTER:
1616      return SafeModeActionProto.SAFEMODE_ENTER;
1617    case SAFEMODE_GET:
1618      return SafeModeActionProto.SAFEMODE_GET;
1619    default:
1620      throw new IllegalArgumentException("Unexpected SafeModeAction :" + a);
1621    }
1622  }
1623  
1624  public static SafeModeAction convert(
1625      ClientNamenodeProtocolProtos.SafeModeActionProto a) {
1626    switch (a) {
1627    case SAFEMODE_LEAVE:
1628      return SafeModeAction.SAFEMODE_LEAVE;
1629    case SAFEMODE_ENTER:
1630      return SafeModeAction.SAFEMODE_ENTER;
1631    case SAFEMODE_GET:
1632      return SafeModeAction.SAFEMODE_GET;
1633    default:
1634      throw new IllegalArgumentException("Unexpected SafeModeAction :" + a);
1635    }
1636  }
1637  
1638  public static RollingUpgradeActionProto convert(RollingUpgradeAction a) {
1639    switch (a) {
1640    case QUERY:
1641      return RollingUpgradeActionProto.QUERY;
1642    case PREPARE:
1643      return RollingUpgradeActionProto.START;
1644    case FINALIZE:
1645      return RollingUpgradeActionProto.FINALIZE;
1646    default:
1647      throw new IllegalArgumentException("Unexpected value: " + a);
1648    }
1649  }
1650  
1651  public static RollingUpgradeAction convert(RollingUpgradeActionProto a) {
1652    switch (a) {
1653    case QUERY:
1654      return RollingUpgradeAction.QUERY;
1655    case START:
1656      return RollingUpgradeAction.PREPARE;
1657    case FINALIZE:
1658      return RollingUpgradeAction.FINALIZE;
1659    default:
1660      throw new IllegalArgumentException("Unexpected value: " + a);
1661    }
1662  }
1663
1664  public static RollingUpgradeStatusProto convertRollingUpgradeStatus(
1665      RollingUpgradeStatus status) {
1666    return RollingUpgradeStatusProto.newBuilder()
1667        .setBlockPoolId(status.getBlockPoolId())
1668        .build();
1669  }
1670
1671  public static RollingUpgradeStatus convert(RollingUpgradeStatusProto proto) {
1672    return new RollingUpgradeStatus(proto.getBlockPoolId());
1673  }
1674
1675  public static RollingUpgradeInfoProto convert(RollingUpgradeInfo info) {
1676    return RollingUpgradeInfoProto.newBuilder()
1677        .setStatus(convertRollingUpgradeStatus(info))
1678        .setCreatedRollbackImages(info.createdRollbackImages())
1679        .setStartTime(info.getStartTime())
1680        .setFinalizeTime(info.getFinalizeTime())
1681        .build();
1682  }
1683
1684  public static RollingUpgradeInfo convert(RollingUpgradeInfoProto proto) {
1685    RollingUpgradeStatusProto status = proto.getStatus();
1686    return new RollingUpgradeInfo(status.getBlockPoolId(),
1687        proto.getCreatedRollbackImages(),
1688        proto.getStartTime(), proto.getFinalizeTime());
1689  }
1690
1691  public static CorruptFileBlocks convert(CorruptFileBlocksProto c) {
1692    if (c == null)
1693      return null;
1694    List<String> fileList = c.getFilesList();
1695    return new CorruptFileBlocks(fileList.toArray(new String[fileList.size()]),
1696        c.getCookie());
1697  }
1698
1699  public static CorruptFileBlocksProto convert(CorruptFileBlocks c) {
1700    if (c == null)
1701      return null;
1702    return CorruptFileBlocksProto.newBuilder().
1703        addAllFiles(Arrays.asList(c.getFiles())).
1704        setCookie(c.getCookie()).
1705        build();
1706  }
1707  
1708  public static ContentSummary convert(ContentSummaryProto cs) {
1709    if (cs == null) return null;
1710    return new ContentSummary(
1711      cs.getLength(), cs.getFileCount(), cs.getDirectoryCount(), cs.getQuota(),
1712      cs.getSpaceConsumed(), cs.getSpaceQuota());
1713  }
1714  
1715  public static ContentSummaryProto convert(ContentSummary cs) {
1716    if (cs == null) return null;
1717    return ContentSummaryProto.newBuilder().
1718        setLength(cs.getLength()).
1719        setFileCount(cs.getFileCount()).
1720        setDirectoryCount(cs.getDirectoryCount()).
1721        setQuota(cs.getQuota()).
1722        setSpaceConsumed(cs.getSpaceConsumed()).
1723        setSpaceQuota(cs.getSpaceQuota()).
1724        build();
1725  }
1726
1727  public static NNHAStatusHeartbeat convert(NNHAStatusHeartbeatProto s) {
1728    if (s == null) return null;
1729    switch (s.getState()) {
1730    case ACTIVE:
1731      return new NNHAStatusHeartbeat(HAServiceState.ACTIVE, s.getTxid());
1732    case STANDBY:
1733      return new NNHAStatusHeartbeat(HAServiceState.STANDBY, s.getTxid());
1734    default:
1735      throw new IllegalArgumentException("Unexpected NNHAStatusHeartbeat.State:" + s.getState());
1736    }
1737  }
1738
1739  public static NNHAStatusHeartbeatProto convert(NNHAStatusHeartbeat hb) {
1740    if (hb == null) return null;
1741    NNHAStatusHeartbeatProto.Builder builder =
1742      NNHAStatusHeartbeatProto.newBuilder();
1743    switch (hb.getState()) {
1744      case ACTIVE:
1745        builder.setState(NNHAStatusHeartbeatProto.State.ACTIVE);
1746        break;
1747      case STANDBY:
1748        builder.setState(NNHAStatusHeartbeatProto.State.STANDBY);
1749        break;
1750      default:
1751        throw new IllegalArgumentException("Unexpected NNHAStatusHeartbeat.State:" +
1752            hb.getState());
1753    }
1754    builder.setTxid(hb.getTxId());
1755    return builder.build();
1756  }
1757
1758  public static DatanodeStorageProto convert(DatanodeStorage s) {
1759    return DatanodeStorageProto.newBuilder()
1760        .setState(PBHelper.convertState(s.getState()))
1761        .setStorageType(PBHelper.convertStorageType(s.getStorageType()))
1762        .setStorageUuid(s.getStorageID()).build();
1763  }
1764
1765  private static StorageState convertState(State state) {
1766    switch(state) {
1767    case READ_ONLY_SHARED:
1768      return StorageState.READ_ONLY_SHARED;
1769    case NORMAL:
1770    default:
1771      return StorageState.NORMAL;
1772    }
1773  }
1774
1775  public static List<StorageTypeProto> convertStorageTypes(
1776      StorageType[] types) {
1777    return convertStorageTypes(types, 0);
1778  }
1779
1780  public static List<StorageTypeProto> convertStorageTypes(
1781      StorageType[] types, int startIdx) {
1782    if (types == null) {
1783      return null;
1784    }
1785    final List<StorageTypeProto> protos = new ArrayList<StorageTypeProto>(
1786        types.length);
1787    for (int i = startIdx; i < types.length; ++i) {
1788      protos.add(convertStorageType(types[i]));
1789    }
1790    return protos; 
1791  }
1792
1793  public static StorageTypeProto convertStorageType(StorageType type) {
1794    switch(type) {
1795    case DISK:
1796      return StorageTypeProto.DISK;
1797    case SSD:
1798      return StorageTypeProto.SSD;
1799    case ARCHIVE:
1800      return StorageTypeProto.ARCHIVE;
1801    case RAM_DISK:
1802      return StorageTypeProto.RAM_DISK;
1803    default:
1804      throw new IllegalStateException(
1805          "BUG: StorageType not found, type=" + type);
1806    }
1807  }
1808
1809  public static DatanodeStorage convert(DatanodeStorageProto s) {
1810    return new DatanodeStorage(s.getStorageUuid(),
1811                               PBHelper.convertState(s.getState()),
1812                               PBHelper.convertStorageType(s.getStorageType()));
1813  }
1814
1815  private static State convertState(StorageState state) {
1816    switch(state) {
1817    case READ_ONLY_SHARED:
1818      return DatanodeStorage.State.READ_ONLY_SHARED;
1819    case NORMAL:
1820    default:
1821      return DatanodeStorage.State.NORMAL;
1822    }
1823  }
1824
1825  public static StorageType convertStorageType(StorageTypeProto type) {
1826    switch(type) {
1827      case DISK:
1828        return StorageType.DISK;
1829      case SSD:
1830        return StorageType.SSD;
1831      case ARCHIVE:
1832        return StorageType.ARCHIVE;
1833      case RAM_DISK:
1834        return StorageType.RAM_DISK;
1835      default:
1836        throw new IllegalStateException(
1837            "BUG: StorageTypeProto not found, type=" + type);
1838    }
1839  }
1840
1841  public static StorageType[] convertStorageTypes(
1842      List<StorageTypeProto> storageTypesList, int expectedSize) {
1843    final StorageType[] storageTypes = new StorageType[expectedSize];
1844    if (storageTypesList.size() != expectedSize) { // missing storage types
1845      Preconditions.checkState(storageTypesList.isEmpty());
1846      Arrays.fill(storageTypes, StorageType.DEFAULT);
1847    } else {
1848      for (int i = 0; i < storageTypes.length; ++i) {
1849        storageTypes[i] = convertStorageType(storageTypesList.get(i));
1850      }
1851    }
1852    return storageTypes;
1853  }
1854
1855  public static StorageReportProto convert(StorageReport r) {
1856    StorageReportProto.Builder builder = StorageReportProto.newBuilder()
1857        .setBlockPoolUsed(r.getBlockPoolUsed()).setCapacity(r.getCapacity())
1858        .setDfsUsed(r.getDfsUsed()).setRemaining(r.getRemaining())
1859        .setStorageUuid(r.getStorage().getStorageID())
1860        .setStorage(convert(r.getStorage()));
1861    return builder.build();
1862  }
1863
1864  public static StorageReport convert(StorageReportProto p) {
1865    return new StorageReport(
1866        p.hasStorage() ?
1867            convert(p.getStorage()) :
1868            new DatanodeStorage(p.getStorageUuid()),
1869        p.getFailed(), p.getCapacity(), p.getDfsUsed(), p.getRemaining(),
1870        p.getBlockPoolUsed());
1871  }
1872
1873  public static StorageReport[] convertStorageReports(
1874      List<StorageReportProto> list) {
1875    final StorageReport[] report = new StorageReport[list.size()];
1876    for (int i = 0; i < report.length; i++) {
1877      report[i] = convert(list.get(i));
1878    }
1879    return report;
1880  }
1881
1882  public static List<StorageReportProto> convertStorageReports(StorageReport[] storages) {
1883    final List<StorageReportProto> protos = new ArrayList<StorageReportProto>(
1884        storages.length);
1885    for(int i = 0; i < storages.length; i++) {
1886      protos.add(convert(storages[i]));
1887    }
1888    return protos;
1889  }
1890
1891  public static JournalInfo convert(JournalInfoProto info) {
1892    int lv = info.hasLayoutVersion() ? info.getLayoutVersion() : 0;
1893    int nsID = info.hasNamespaceID() ? info.getNamespaceID() : 0;
1894    return new JournalInfo(lv, info.getClusterID(), nsID);
1895  }
1896
1897  /**
1898   * Method used for converting {@link JournalInfoProto} sent from Namenode
1899   * to Journal receivers to {@link NamenodeRegistration}.
1900   */
1901  public static JournalInfoProto convert(JournalInfo j) {
1902    return JournalInfoProto.newBuilder().setClusterID(j.getClusterId())
1903        .setLayoutVersion(j.getLayoutVersion())
1904        .setNamespaceID(j.getNamespaceId()).build();
1905  } 
1906  
1907  public static SnapshottableDirectoryStatus[] convert(
1908      SnapshottableDirectoryListingProto sdlp) {
1909    if (sdlp == null)
1910      return null;
1911    List<SnapshottableDirectoryStatusProto> list = sdlp
1912        .getSnapshottableDirListingList();
1913    if (list.isEmpty()) {
1914      return new SnapshottableDirectoryStatus[0];
1915    } else {
1916      SnapshottableDirectoryStatus[] result = 
1917          new SnapshottableDirectoryStatus[list.size()];
1918      for (int i = 0; i < list.size(); i++) {
1919        result[i] = PBHelper.convert(list.get(i));
1920      }
1921      return result;
1922    }
1923  }
1924  
1925  public static SnapshottableDirectoryListingProto convert(
1926      SnapshottableDirectoryStatus[] status) {
1927    if (status == null)
1928      return null;
1929    SnapshottableDirectoryStatusProto[] protos = 
1930        new SnapshottableDirectoryStatusProto[status.length];
1931    for (int i = 0; i < status.length; i++) {
1932      protos[i] = PBHelper.convert(status[i]);
1933    }
1934    List<SnapshottableDirectoryStatusProto> protoList = Arrays.asList(protos);
1935    return SnapshottableDirectoryListingProto.newBuilder()
1936        .addAllSnapshottableDirListing(protoList).build();
1937  }
1938  
1939  public static DiffReportEntry convert(SnapshotDiffReportEntryProto entry) {
1940    if (entry == null) {
1941      return null;
1942    }
1943    DiffType type = DiffType.getTypeFromLabel(entry
1944        .getModificationLabel());
1945    return type == null ? null : new DiffReportEntry(type, entry.getFullpath()
1946        .toByteArray(), entry.hasTargetPath() ? entry.getTargetPath()
1947        .toByteArray() : null);
1948  }
1949  
1950  public static SnapshotDiffReportEntryProto convert(DiffReportEntry entry) {
1951    if (entry == null) {
1952      return null;
1953    }
1954    ByteString sourcePath = ByteString
1955        .copyFrom(entry.getSourcePath() == null ? DFSUtil.EMPTY_BYTES : entry
1956            .getSourcePath());
1957    String modification = entry.getType().getLabel();
1958    SnapshotDiffReportEntryProto.Builder builder = SnapshotDiffReportEntryProto
1959        .newBuilder().setFullpath(sourcePath)
1960        .setModificationLabel(modification);
1961    if (entry.getType() == DiffType.RENAME) {
1962      ByteString targetPath = ByteString
1963          .copyFrom(entry.getTargetPath() == null ? DFSUtil.EMPTY_BYTES : entry
1964              .getTargetPath());
1965      builder.setTargetPath(targetPath);
1966    }
1967    return builder.build();
1968  }
1969  
1970  public static SnapshotDiffReport convert(SnapshotDiffReportProto reportProto) {
1971    if (reportProto == null) {
1972      return null;
1973    }
1974    String snapshotDir = reportProto.getSnapshotRoot();
1975    String fromSnapshot = reportProto.getFromSnapshot();
1976    String toSnapshot = reportProto.getToSnapshot();
1977    List<SnapshotDiffReportEntryProto> list = reportProto
1978        .getDiffReportEntriesList();
1979    List<DiffReportEntry> entries = new ArrayList<DiffReportEntry>();
1980    for (SnapshotDiffReportEntryProto entryProto : list) {
1981      DiffReportEntry entry = convert(entryProto);
1982      if (entry != null)
1983        entries.add(entry);
1984    }
1985    return new SnapshotDiffReport(snapshotDir, fromSnapshot, toSnapshot,
1986        entries);
1987  }
1988  
1989  public static SnapshotDiffReportProto convert(SnapshotDiffReport report) {
1990    if (report == null) {
1991      return null;
1992    }
1993    List<DiffReportEntry> entries = report.getDiffList();
1994    List<SnapshotDiffReportEntryProto> entryProtos = 
1995        new ArrayList<SnapshotDiffReportEntryProto>();
1996    for (DiffReportEntry entry : entries) {
1997      SnapshotDiffReportEntryProto entryProto = convert(entry);
1998      if (entryProto != null)
1999        entryProtos.add(entryProto);
2000    }
2001    
2002    SnapshotDiffReportProto reportProto = SnapshotDiffReportProto.newBuilder()
2003        .setSnapshotRoot(report.getSnapshotRoot())
2004        .setFromSnapshot(report.getFromSnapshot())
2005        .setToSnapshot(report.getLaterSnapshotName())
2006        .addAllDiffReportEntries(entryProtos).build();
2007    return reportProto;
2008  }
2009
2010  public static DataChecksum.Type convert(HdfsProtos.ChecksumTypeProto type) {
2011    return DataChecksum.Type.valueOf(type.getNumber());
2012  }
2013
2014  public static CacheDirectiveInfoProto convert
2015      (CacheDirectiveInfo info) {
2016    CacheDirectiveInfoProto.Builder builder = 
2017        CacheDirectiveInfoProto.newBuilder();
2018    if (info.getId() != null) {
2019      builder.setId(info.getId());
2020    }
2021    if (info.getPath() != null) {
2022      builder.setPath(info.getPath().toUri().getPath());
2023    }
2024    if (info.getReplication() != null) {
2025      builder.setReplication(info.getReplication());
2026    }
2027    if (info.getPool() != null) {
2028      builder.setPool(info.getPool());
2029    }
2030    if (info.getExpiration() != null) {
2031      builder.setExpiration(convert(info.getExpiration()));
2032    }
2033    return builder.build();
2034  }
2035
2036  public static CacheDirectiveInfo convert
2037      (CacheDirectiveInfoProto proto) {
2038    CacheDirectiveInfo.Builder builder =
2039        new CacheDirectiveInfo.Builder();
2040    if (proto.hasId()) {
2041      builder.setId(proto.getId());
2042    }
2043    if (proto.hasPath()) {
2044      builder.setPath(new Path(proto.getPath()));
2045    }
2046    if (proto.hasReplication()) {
2047      builder.setReplication(Shorts.checkedCast(
2048          proto.getReplication()));
2049    }
2050    if (proto.hasPool()) {
2051      builder.setPool(proto.getPool());
2052    }
2053    if (proto.hasExpiration()) {
2054      builder.setExpiration(convert(proto.getExpiration()));
2055    }
2056    return builder.build();
2057  }
2058
2059  public static CacheDirectiveInfoExpirationProto convert(
2060      CacheDirectiveInfo.Expiration expiration) {
2061    return CacheDirectiveInfoExpirationProto.newBuilder()
2062        .setIsRelative(expiration.isRelative())
2063        .setMillis(expiration.getMillis())
2064        .build();
2065  }
2066
2067  public static CacheDirectiveInfo.Expiration convert(
2068      CacheDirectiveInfoExpirationProto proto) {
2069    if (proto.getIsRelative()) {
2070      return CacheDirectiveInfo.Expiration.newRelative(proto.getMillis());
2071    }
2072    return CacheDirectiveInfo.Expiration.newAbsolute(proto.getMillis());
2073  }
2074
2075  public static CacheDirectiveStatsProto convert(CacheDirectiveStats stats) {
2076    CacheDirectiveStatsProto.Builder builder = 
2077        CacheDirectiveStatsProto.newBuilder();
2078    builder.setBytesNeeded(stats.getBytesNeeded());
2079    builder.setBytesCached(stats.getBytesCached());
2080    builder.setFilesNeeded(stats.getFilesNeeded());
2081    builder.setFilesCached(stats.getFilesCached());
2082    builder.setHasExpired(stats.hasExpired());
2083    return builder.build();
2084  }
2085  
2086  public static CacheDirectiveStats convert(CacheDirectiveStatsProto proto) {
2087    CacheDirectiveStats.Builder builder = new CacheDirectiveStats.Builder();
2088    builder.setBytesNeeded(proto.getBytesNeeded());
2089    builder.setBytesCached(proto.getBytesCached());
2090    builder.setFilesNeeded(proto.getFilesNeeded());
2091    builder.setFilesCached(proto.getFilesCached());
2092    builder.setHasExpired(proto.getHasExpired());
2093    return builder.build();
2094  }
2095
2096  public static CacheDirectiveEntryProto convert(CacheDirectiveEntry entry) {
2097    CacheDirectiveEntryProto.Builder builder = 
2098        CacheDirectiveEntryProto.newBuilder();
2099    builder.setInfo(PBHelper.convert(entry.getInfo()));
2100    builder.setStats(PBHelper.convert(entry.getStats()));
2101    return builder.build();
2102  }
2103  
2104  public static CacheDirectiveEntry convert(CacheDirectiveEntryProto proto) {
2105    CacheDirectiveInfo info = PBHelper.convert(proto.getInfo());
2106    CacheDirectiveStats stats = PBHelper.convert(proto.getStats());
2107    return new CacheDirectiveEntry(info, stats);
2108  }
2109
2110  public static CachePoolInfoProto convert(CachePoolInfo info) {
2111    CachePoolInfoProto.Builder builder = CachePoolInfoProto.newBuilder();
2112    builder.setPoolName(info.getPoolName());
2113    if (info.getOwnerName() != null) {
2114      builder.setOwnerName(info.getOwnerName());
2115    }
2116    if (info.getGroupName() != null) {
2117      builder.setGroupName(info.getGroupName());
2118    }
2119    if (info.getMode() != null) {
2120      builder.setMode(info.getMode().toShort());
2121    }
2122    if (info.getLimit() != null) {
2123      builder.setLimit(info.getLimit());
2124    }
2125    if (info.getMaxRelativeExpiryMs() != null) {
2126      builder.setMaxRelativeExpiry(info.getMaxRelativeExpiryMs());
2127    }
2128    return builder.build();
2129  }
2130
2131  public static CachePoolInfo convert (CachePoolInfoProto proto) {
2132    // Pool name is a required field, the rest are optional
2133    String poolName = checkNotNull(proto.getPoolName());
2134    CachePoolInfo info = new CachePoolInfo(poolName);
2135    if (proto.hasOwnerName()) {
2136        info.setOwnerName(proto.getOwnerName());
2137    }
2138    if (proto.hasGroupName()) {
2139      info.setGroupName(proto.getGroupName());
2140    }
2141    if (proto.hasMode()) {
2142      info.setMode(new FsPermission((short)proto.getMode()));
2143    }
2144    if (proto.hasLimit())  {
2145      info.setLimit(proto.getLimit());
2146    }
2147    if (proto.hasMaxRelativeExpiry()) {
2148      info.setMaxRelativeExpiryMs(proto.getMaxRelativeExpiry());
2149    }
2150    return info;
2151  }
2152
2153  public static CachePoolStatsProto convert(CachePoolStats stats) {
2154    CachePoolStatsProto.Builder builder = CachePoolStatsProto.newBuilder();
2155    builder.setBytesNeeded(stats.getBytesNeeded());
2156    builder.setBytesCached(stats.getBytesCached());
2157    builder.setBytesOverlimit(stats.getBytesOverlimit());
2158    builder.setFilesNeeded(stats.getFilesNeeded());
2159    builder.setFilesCached(stats.getFilesCached());
2160    return builder.build();
2161  }
2162
2163  public static CachePoolStats convert (CachePoolStatsProto proto) {
2164    CachePoolStats.Builder builder = new CachePoolStats.Builder();
2165    builder.setBytesNeeded(proto.getBytesNeeded());
2166    builder.setBytesCached(proto.getBytesCached());
2167    builder.setBytesOverlimit(proto.getBytesOverlimit());
2168    builder.setFilesNeeded(proto.getFilesNeeded());
2169    builder.setFilesCached(proto.getFilesCached());
2170    return builder.build();
2171  }
2172
2173  public static CachePoolEntryProto convert(CachePoolEntry entry) {
2174    CachePoolEntryProto.Builder builder = CachePoolEntryProto.newBuilder();
2175    builder.setInfo(PBHelper.convert(entry.getInfo()));
2176    builder.setStats(PBHelper.convert(entry.getStats()));
2177    return builder.build();
2178  }
2179
2180  public static CachePoolEntry convert (CachePoolEntryProto proto) {
2181    CachePoolInfo info = PBHelper.convert(proto.getInfo());
2182    CachePoolStats stats = PBHelper.convert(proto.getStats());
2183    return new CachePoolEntry(info, stats);
2184  }
2185  
2186  public static HdfsProtos.ChecksumTypeProto convert(DataChecksum.Type type) {
2187    return HdfsProtos.ChecksumTypeProto.valueOf(type.id);
2188  }
2189
2190  public static DatanodeLocalInfoProto convert(DatanodeLocalInfo info) {
2191    DatanodeLocalInfoProto.Builder builder = DatanodeLocalInfoProto.newBuilder();
2192    builder.setSoftwareVersion(info.getSoftwareVersion());
2193    builder.setConfigVersion(info.getConfigVersion());
2194    builder.setUptime(info.getUptime());
2195    return builder.build();
2196  }
2197
2198  public static DatanodeLocalInfo convert(DatanodeLocalInfoProto proto) {
2199    return new DatanodeLocalInfo(proto.getSoftwareVersion(),
2200        proto.getConfigVersion(), proto.getUptime());
2201  }
2202
2203  public static InputStream vintPrefixed(final InputStream input)
2204      throws IOException {
2205    final int firstByte = input.read();
2206    if (firstByte == -1) {
2207      throw new EOFException("Premature EOF: no length prefix available");
2208    }
2209
2210    int size = CodedInputStream.readRawVarint32(firstByte, input);
2211    assert size >= 0;
2212    return new ExactSizeInputStream(input, size);
2213  }
2214
2215  private static AclEntryScopeProto convert(AclEntryScope v) {
2216    return AclEntryScopeProto.valueOf(v.ordinal());
2217  }
2218
2219  private static AclEntryScope convert(AclEntryScopeProto v) {
2220    return castEnum(v, ACL_ENTRY_SCOPE_VALUES);
2221  }
2222
2223  private static AclEntryTypeProto convert(AclEntryType e) {
2224    return AclEntryTypeProto.valueOf(e.ordinal());
2225  }
2226
2227  private static AclEntryType convert(AclEntryTypeProto v) {
2228    return castEnum(v, ACL_ENTRY_TYPE_VALUES);
2229  }
2230  
2231  private static XAttrNamespaceProto convert(XAttr.NameSpace v) {
2232    return XAttrNamespaceProto.valueOf(v.ordinal());
2233  }
2234  
2235  private static XAttr.NameSpace convert(XAttrNamespaceProto v) {
2236    return castEnum(v, XATTR_NAMESPACE_VALUES);
2237  }
2238
2239  public static FsActionProto convert(FsAction v) {
2240    return FsActionProto.valueOf(v != null ? v.ordinal() : 0);
2241  }
2242
2243  public static FsAction convert(FsActionProto v) {
2244    return castEnum(v, FSACTION_VALUES);
2245  }
2246
2247  public static List<AclEntryProto> convertAclEntryProto(
2248      List<AclEntry> aclSpec) {
2249    ArrayList<AclEntryProto> r = Lists.newArrayListWithCapacity(aclSpec.size());
2250    for (AclEntry e : aclSpec) {
2251      AclEntryProto.Builder builder = AclEntryProto.newBuilder();
2252      builder.setType(convert(e.getType()));
2253      builder.setScope(convert(e.getScope()));
2254      builder.setPermissions(convert(e.getPermission()));
2255      if (e.getName() != null) {
2256        builder.setName(e.getName());
2257      }
2258      r.add(builder.build());
2259    }
2260    return r;
2261  }
2262
2263  public static List<AclEntry> convertAclEntry(List<AclEntryProto> aclSpec) {
2264    ArrayList<AclEntry> r = Lists.newArrayListWithCapacity(aclSpec.size());
2265    for (AclEntryProto e : aclSpec) {
2266      AclEntry.Builder builder = new AclEntry.Builder();
2267      builder.setType(convert(e.getType()));
2268      builder.setScope(convert(e.getScope()));
2269      builder.setPermission(convert(e.getPermissions()));
2270      if (e.hasName()) {
2271        builder.setName(e.getName());
2272      }
2273      r.add(builder.build());
2274    }
2275    return r;
2276  }
2277
2278  public static AclStatus convert(GetAclStatusResponseProto e) {
2279    AclStatusProto r = e.getResult();
2280    return new AclStatus.Builder().owner(r.getOwner()).group(r.getGroup())
2281        .stickyBit(r.getSticky())
2282        .addEntries(convertAclEntry(r.getEntriesList())).build();
2283  }
2284
2285  public static GetAclStatusResponseProto convert(AclStatus e) {
2286    AclStatusProto r = AclStatusProto.newBuilder().setOwner(e.getOwner())
2287        .setGroup(e.getGroup()).setSticky(e.isStickyBit())
2288        .addAllEntries(convertAclEntryProto(e.getEntries())).build();
2289    return GetAclStatusResponseProto.newBuilder().setResult(r).build();
2290  }
2291  
2292  public static XAttrProto convertXAttrProto(XAttr a) {
2293    XAttrProto.Builder builder = XAttrProto.newBuilder();
2294    builder.setNamespace(convert(a.getNameSpace()));
2295    if (a.getName() != null) {
2296      builder.setName(a.getName());
2297    }
2298    if (a.getValue() != null) {
2299      builder.setValue(getByteString(a.getValue()));
2300    }
2301    return builder.build();
2302  }
2303  
2304  public static List<XAttrProto> convertXAttrProto(
2305      List<XAttr> xAttrSpec) {
2306    if (xAttrSpec == null) {
2307      return Lists.newArrayListWithCapacity(0);
2308    }
2309    ArrayList<XAttrProto> xAttrs = Lists.newArrayListWithCapacity(
2310        xAttrSpec.size());
2311    for (XAttr a : xAttrSpec) {
2312      XAttrProto.Builder builder = XAttrProto.newBuilder();
2313      builder.setNamespace(convert(a.getNameSpace()));
2314      if (a.getName() != null) {
2315        builder.setName(a.getName());
2316      }
2317      if (a.getValue() != null) {
2318        builder.setValue(getByteString(a.getValue()));
2319      }
2320      xAttrs.add(builder.build());
2321    }
2322    return xAttrs;
2323  }
2324  
2325  /**
2326   * The flag field in PB is a bitmask whose values are the same a the 
2327   * emum values of XAttrSetFlag
2328   */
2329  public static int convert(EnumSet<XAttrSetFlag> flag) {
2330    int value = 0;
2331    if (flag.contains(XAttrSetFlag.CREATE)) {
2332      value |= XAttrSetFlagProto.XATTR_CREATE.getNumber();
2333    }
2334    if (flag.contains(XAttrSetFlag.REPLACE)) {
2335      value |= XAttrSetFlagProto.XATTR_REPLACE.getNumber();
2336    }
2337    return value;
2338  }
2339 
2340  public static EnumSet<XAttrSetFlag> convert(int flag) {
2341    EnumSet<XAttrSetFlag> result = 
2342        EnumSet.noneOf(XAttrSetFlag.class);
2343    if ((flag & XAttrSetFlagProto.XATTR_CREATE_VALUE) == 
2344        XAttrSetFlagProto.XATTR_CREATE_VALUE) {
2345      result.add(XAttrSetFlag.CREATE);
2346    }
2347    if ((flag & XAttrSetFlagProto.XATTR_REPLACE_VALUE) == 
2348        XAttrSetFlagProto.XATTR_REPLACE_VALUE) {
2349      result.add(XAttrSetFlag.REPLACE);
2350    }
2351    return result;
2352  }
2353  
2354  public static XAttr convertXAttr(XAttrProto a) {
2355    XAttr.Builder builder = new XAttr.Builder();
2356    builder.setNameSpace(convert(a.getNamespace()));
2357    if (a.hasName()) {
2358      builder.setName(a.getName());
2359    }
2360    if (a.hasValue()) {
2361      builder.setValue(a.getValue().toByteArray());
2362    }
2363    return builder.build();
2364  }
2365  
2366  public static List<XAttr> convertXAttrs(List<XAttrProto> xAttrSpec) {
2367    ArrayList<XAttr> xAttrs = Lists.newArrayListWithCapacity(xAttrSpec.size());
2368    for (XAttrProto a : xAttrSpec) {
2369      XAttr.Builder builder = new XAttr.Builder();
2370      builder.setNameSpace(convert(a.getNamespace()));
2371      if (a.hasName()) {
2372        builder.setName(a.getName());
2373      }
2374      if (a.hasValue()) {
2375        builder.setValue(a.getValue().toByteArray());
2376      }
2377      xAttrs.add(builder.build());
2378    }
2379    return xAttrs;
2380  }
2381
2382  public static List<XAttr> convert(GetXAttrsResponseProto a) {
2383    List<XAttrProto> xAttrs = a.getXAttrsList();
2384    return convertXAttrs(xAttrs);
2385  }
2386
2387  public static GetXAttrsResponseProto convertXAttrsResponse(
2388      List<XAttr> xAttrs) {
2389    GetXAttrsResponseProto.Builder builder = GetXAttrsResponseProto
2390        .newBuilder();
2391    if (xAttrs != null) {
2392      builder.addAllXAttrs(convertXAttrProto(xAttrs));
2393    }
2394    return builder.build();
2395  }
2396
2397  public static List<XAttr> convert(ListXAttrsResponseProto a) {
2398    final List<XAttrProto> xAttrs = a.getXAttrsList();
2399    return convertXAttrs(xAttrs);
2400  }
2401
2402  public static ListXAttrsResponseProto convertListXAttrsResponse(
2403    List<XAttr> names) {
2404    ListXAttrsResponseProto.Builder builder =
2405      ListXAttrsResponseProto.newBuilder();
2406    if (names != null) {
2407      builder.addAllXAttrs(convertXAttrProto(names));
2408    }
2409    return builder.build();
2410  }
2411
2412  public static EncryptionZoneProto convert(EncryptionZone zone) {
2413    return EncryptionZoneProto.newBuilder()
2414        .setId(zone.getId())
2415        .setPath(zone.getPath())
2416        .setSuite(convert(zone.getSuite()))
2417        .setCryptoProtocolVersion(convert(zone.getVersion()))
2418        .setKeyName(zone.getKeyName())
2419        .build();
2420  }
2421
2422  public static EncryptionZone convert(EncryptionZoneProto proto) {
2423    return new EncryptionZone(proto.getId(), proto.getPath(),
2424        convert(proto.getSuite()), convert(proto.getCryptoProtocolVersion()),
2425        proto.getKeyName());
2426  }
2427
2428  public static ShortCircuitShmSlotProto convert(SlotId slotId) {
2429    return ShortCircuitShmSlotProto.newBuilder().
2430        setShmId(convert(slotId.getShmId())).
2431        setSlotIdx(slotId.getSlotIdx()).
2432        build();
2433  }
2434
2435  public static ShortCircuitShmIdProto convert(ShmId shmId) {
2436    return ShortCircuitShmIdProto.newBuilder().
2437        setHi(shmId.getHi()).
2438        setLo(shmId.getLo()).
2439        build();
2440
2441  }
2442
2443  public static SlotId convert(ShortCircuitShmSlotProto slotId) {
2444    return new SlotId(PBHelper.convert(slotId.getShmId()),
2445        slotId.getSlotIdx());
2446  }
2447
2448  public static ShmId convert(ShortCircuitShmIdProto shmId) {
2449    return new ShmId(shmId.getHi(), shmId.getLo());
2450  }
2451
2452  private static Event.CreateEvent.INodeType createTypeConvert(InotifyProtos.INodeType
2453      type) {
2454    switch (type) {
2455    case I_TYPE_DIRECTORY:
2456      return Event.CreateEvent.INodeType.DIRECTORY;
2457    case I_TYPE_FILE:
2458      return Event.CreateEvent.INodeType.FILE;
2459    case I_TYPE_SYMLINK:
2460      return Event.CreateEvent.INodeType.SYMLINK;
2461    default:
2462      return null;
2463    }
2464  }
2465
2466  private static InotifyProtos.MetadataUpdateType metadataUpdateTypeConvert(
2467      Event.MetadataUpdateEvent.MetadataType type) {
2468    switch (type) {
2469    case TIMES:
2470      return InotifyProtos.MetadataUpdateType.META_TYPE_TIMES;
2471    case REPLICATION:
2472      return InotifyProtos.MetadataUpdateType.META_TYPE_REPLICATION;
2473    case OWNER:
2474      return InotifyProtos.MetadataUpdateType.META_TYPE_OWNER;
2475    case PERMS:
2476      return InotifyProtos.MetadataUpdateType.META_TYPE_PERMS;
2477    case ACLS:
2478      return InotifyProtos.MetadataUpdateType.META_TYPE_ACLS;
2479    case XATTRS:
2480      return InotifyProtos.MetadataUpdateType.META_TYPE_XATTRS;
2481    default:
2482      return null;
2483    }
2484  }
2485
2486  private static Event.MetadataUpdateEvent.MetadataType metadataUpdateTypeConvert(
2487      InotifyProtos.MetadataUpdateType type) {
2488    switch (type) {
2489    case META_TYPE_TIMES:
2490      return Event.MetadataUpdateEvent.MetadataType.TIMES;
2491    case META_TYPE_REPLICATION:
2492      return Event.MetadataUpdateEvent.MetadataType.REPLICATION;
2493    case META_TYPE_OWNER:
2494      return Event.MetadataUpdateEvent.MetadataType.OWNER;
2495    case META_TYPE_PERMS:
2496      return Event.MetadataUpdateEvent.MetadataType.PERMS;
2497    case META_TYPE_ACLS:
2498      return Event.MetadataUpdateEvent.MetadataType.ACLS;
2499    case META_TYPE_XATTRS:
2500      return Event.MetadataUpdateEvent.MetadataType.XATTRS;
2501    default:
2502      return null;
2503    }
2504  }
2505
2506  private static InotifyProtos.INodeType createTypeConvert(Event.CreateEvent.INodeType
2507      type) {
2508    switch (type) {
2509    case DIRECTORY:
2510      return InotifyProtos.INodeType.I_TYPE_DIRECTORY;
2511    case FILE:
2512      return InotifyProtos.INodeType.I_TYPE_FILE;
2513    case SYMLINK:
2514      return InotifyProtos.INodeType.I_TYPE_SYMLINK;
2515    default:
2516      return null;
2517    }
2518  }
2519
2520  public static EventBatchList convert(GetEditsFromTxidResponseProto resp) throws
2521    IOException {
2522    final InotifyProtos.EventsListProto list = resp.getEventsList();
2523    final long firstTxid = list.getFirstTxid();
2524    final long lastTxid = list.getLastTxid();
2525
2526    List<EventBatch> batches = Lists.newArrayList();
2527    if (list.getEventsList().size() > 0) {
2528      throw new IOException("Can't handle old inotify server response.");
2529    }
2530    for (InotifyProtos.EventBatchProto bp : list.getBatchList()) {
2531      long txid = bp.getTxid();
2532      if ((txid != -1) && ((txid < firstTxid) || (txid > lastTxid))) {
2533        throw new IOException("Error converting TxidResponseProto: got a " +
2534            "transaction id " + txid + " that was outside the range of [" +
2535            firstTxid + ", " + lastTxid + "].");
2536      }
2537      List<Event> events = Lists.newArrayList();
2538      for (InotifyProtos.EventProto p : bp.getEventsList()) {
2539        switch (p.getType()) {
2540          case EVENT_CLOSE:
2541            InotifyProtos.CloseEventProto close =
2542                InotifyProtos.CloseEventProto.parseFrom(p.getContents());
2543            events.add(new Event.CloseEvent(close.getPath(),
2544                close.getFileSize(), close.getTimestamp()));
2545            break;
2546          case EVENT_CREATE:
2547            InotifyProtos.CreateEventProto create =
2548                InotifyProtos.CreateEventProto.parseFrom(p.getContents());
2549            events.add(new Event.CreateEvent.Builder()
2550                .iNodeType(createTypeConvert(create.getType()))
2551                .path(create.getPath())
2552                .ctime(create.getCtime())
2553                .ownerName(create.getOwnerName())
2554                .groupName(create.getGroupName())
2555                .perms(convert(create.getPerms()))
2556                .replication(create.getReplication())
2557                .symlinkTarget(create.getSymlinkTarget().isEmpty() ? null :
2558                    create.getSymlinkTarget())
2559                .overwrite(create.getOverwrite()).build());
2560            break;
2561          case EVENT_METADATA:
2562            InotifyProtos.MetadataUpdateEventProto meta =
2563                InotifyProtos.MetadataUpdateEventProto.parseFrom(p.getContents());
2564            events.add(new Event.MetadataUpdateEvent.Builder()
2565                .path(meta.getPath())
2566                .metadataType(metadataUpdateTypeConvert(meta.getType()))
2567                .mtime(meta.getMtime())
2568                .atime(meta.getAtime())
2569                .replication(meta.getReplication())
2570                .ownerName(
2571                    meta.getOwnerName().isEmpty() ? null : meta.getOwnerName())
2572                .groupName(
2573                    meta.getGroupName().isEmpty() ? null : meta.getGroupName())
2574                .perms(meta.hasPerms() ? convert(meta.getPerms()) : null)
2575                .acls(meta.getAclsList().isEmpty() ? null : convertAclEntry(
2576                    meta.getAclsList()))
2577                .xAttrs(meta.getXAttrsList().isEmpty() ? null : convertXAttrs(
2578                    meta.getXAttrsList()))
2579                .xAttrsRemoved(meta.getXAttrsRemoved())
2580                .build());
2581            break;
2582          case EVENT_RENAME:
2583            InotifyProtos.RenameEventProto rename =
2584                InotifyProtos.RenameEventProto.parseFrom(p.getContents());
2585            events.add(new Event.RenameEvent(rename.getSrcPath(),
2586                rename.getDestPath(), rename.getTimestamp()));
2587            break;
2588          case EVENT_APPEND:
2589            InotifyProtos.AppendEventProto reopen =
2590                InotifyProtos.AppendEventProto.parseFrom(p.getContents());
2591            events.add(new Event.AppendEvent(reopen.getPath()));
2592            break;
2593          case EVENT_UNLINK:
2594            InotifyProtos.UnlinkEventProto unlink =
2595                InotifyProtos.UnlinkEventProto.parseFrom(p.getContents());
2596            events.add(new Event.UnlinkEvent(unlink.getPath(),
2597                unlink.getTimestamp()));
2598            break;
2599          default:
2600            throw new RuntimeException("Unexpected inotify event type: " +
2601                p.getType());
2602        }
2603      }
2604      batches.add(new EventBatch(txid, events.toArray(new Event[0])));
2605    }
2606    return new EventBatchList(batches, resp.getEventsList().getFirstTxid(),
2607        resp.getEventsList().getLastTxid(), resp.getEventsList().getSyncTxid());
2608  }
2609
2610  public static GetEditsFromTxidResponseProto convertEditsResponse(EventBatchList el) {
2611    InotifyProtos.EventsListProto.Builder builder =
2612        InotifyProtos.EventsListProto.newBuilder();
2613    for (EventBatch b : el.getBatches()) {
2614      List<InotifyProtos.EventProto> events = Lists.newArrayList();
2615      for (Event e : b.getEvents()) {
2616        switch (e.getEventType()) {
2617          case CLOSE:
2618            Event.CloseEvent ce = (Event.CloseEvent) e;
2619            events.add(InotifyProtos.EventProto.newBuilder()
2620                .setType(InotifyProtos.EventType.EVENT_CLOSE)
2621                .setContents(
2622                    InotifyProtos.CloseEventProto.newBuilder()
2623                        .setPath(ce.getPath())
2624                        .setFileSize(ce.getFileSize())
2625                        .setTimestamp(ce.getTimestamp()).build().toByteString()
2626                ).build());
2627            break;
2628          case CREATE:
2629            Event.CreateEvent ce2 = (Event.CreateEvent) e;
2630            events.add(InotifyProtos.EventProto.newBuilder()
2631                .setType(InotifyProtos.EventType.EVENT_CREATE)
2632                .setContents(
2633                    InotifyProtos.CreateEventProto.newBuilder()
2634                        .setType(createTypeConvert(ce2.getiNodeType()))
2635                        .setPath(ce2.getPath())
2636                        .setCtime(ce2.getCtime())
2637                        .setOwnerName(ce2.getOwnerName())
2638                        .setGroupName(ce2.getGroupName())
2639                        .setPerms(convert(ce2.getPerms()))
2640                        .setReplication(ce2.getReplication())
2641                        .setSymlinkTarget(ce2.getSymlinkTarget() == null ?
2642                            "" : ce2.getSymlinkTarget())
2643                        .setOverwrite(ce2.getOverwrite()).build().toByteString()
2644                ).build());
2645            break;
2646          case METADATA:
2647            Event.MetadataUpdateEvent me = (Event.MetadataUpdateEvent) e;
2648            InotifyProtos.MetadataUpdateEventProto.Builder metaB =
2649                InotifyProtos.MetadataUpdateEventProto.newBuilder()
2650                    .setPath(me.getPath())
2651                    .setType(metadataUpdateTypeConvert(me.getMetadataType()))
2652                    .setMtime(me.getMtime())
2653                    .setAtime(me.getAtime())
2654                    .setReplication(me.getReplication())
2655                    .setOwnerName(me.getOwnerName() == null ? "" :
2656                        me.getOwnerName())
2657                    .setGroupName(me.getGroupName() == null ? "" :
2658                        me.getGroupName())
2659                    .addAllAcls(me.getAcls() == null ?
2660                        Lists.<AclEntryProto>newArrayList() :
2661                        convertAclEntryProto(me.getAcls()))
2662                    .addAllXAttrs(me.getxAttrs() == null ?
2663                        Lists.<XAttrProto>newArrayList() :
2664                        convertXAttrProto(me.getxAttrs()))
2665                    .setXAttrsRemoved(me.isxAttrsRemoved());
2666            if (me.getPerms() != null) {
2667              metaB.setPerms(convert(me.getPerms()));
2668            }
2669            events.add(InotifyProtos.EventProto.newBuilder()
2670                .setType(InotifyProtos.EventType.EVENT_METADATA)
2671                .setContents(metaB.build().toByteString())
2672                .build());
2673            break;
2674          case RENAME:
2675            Event.RenameEvent re = (Event.RenameEvent) e;
2676            events.add(InotifyProtos.EventProto.newBuilder()
2677                .setType(InotifyProtos.EventType.EVENT_RENAME)
2678                .setContents(
2679                    InotifyProtos.RenameEventProto.newBuilder()
2680                        .setSrcPath(re.getSrcPath())
2681                        .setDestPath(re.getDstPath())
2682                        .setTimestamp(re.getTimestamp()).build().toByteString()
2683                ).build());
2684            break;
2685          case APPEND:
2686            Event.AppendEvent re2 = (Event.AppendEvent) e;
2687            events.add(InotifyProtos.EventProto.newBuilder()
2688                .setType(InotifyProtos.EventType.EVENT_APPEND)
2689                .setContents(
2690                    InotifyProtos.AppendEventProto.newBuilder()
2691                        .setPath(re2.getPath()).build().toByteString()
2692                ).build());
2693            break;
2694          case UNLINK:
2695            Event.UnlinkEvent ue = (Event.UnlinkEvent) e;
2696            events.add(InotifyProtos.EventProto.newBuilder()
2697                .setType(InotifyProtos.EventType.EVENT_UNLINK)
2698                .setContents(
2699                    InotifyProtos.UnlinkEventProto.newBuilder()
2700                        .setPath(ue.getPath())
2701                        .setTimestamp(ue.getTimestamp()).build().toByteString()
2702                ).build());
2703            break;
2704          default:
2705            throw new RuntimeException("Unexpected inotify event: " + e);
2706        }
2707      }
2708      builder.addBatch(InotifyProtos.EventBatchProto.newBuilder().
2709          setTxid(b.getTxid()).
2710          addAllEvents(events));
2711    }
2712    builder.setFirstTxid(el.getFirstTxid());
2713    builder.setLastTxid(el.getLastTxid());
2714    builder.setSyncTxid(el.getSyncTxid());
2715    return GetEditsFromTxidResponseProto.newBuilder().setEventsList(
2716        builder.build()).build();
2717  }
2718  
2719  public static CipherOptionProto convert(CipherOption option) {
2720    if (option != null) {
2721      CipherOptionProto.Builder builder = CipherOptionProto.
2722          newBuilder();
2723      if (option.getCipherSuite() != null) {
2724        builder.setSuite(convert(option.getCipherSuite()));
2725      }
2726      if (option.getInKey() != null) {
2727        builder.setInKey(ByteString.copyFrom(option.getInKey()));
2728      }
2729      if (option.getInIv() != null) {
2730        builder.setInIv(ByteString.copyFrom(option.getInIv()));
2731      }
2732      if (option.getOutKey() != null) {
2733        builder.setOutKey(ByteString.copyFrom(option.getOutKey()));
2734      }
2735      if (option.getOutIv() != null) {
2736        builder.setOutIv(ByteString.copyFrom(option.getOutIv()));
2737      }
2738      return builder.build();
2739    }
2740    return null;
2741  }
2742  
2743  public static CipherOption convert(CipherOptionProto proto) {
2744    if (proto != null) {
2745      CipherSuite suite = null;
2746      if (proto.getSuite() != null) {
2747        suite = convert(proto.getSuite());
2748      }
2749      byte[] inKey = null;
2750      if (proto.getInKey() != null) {
2751        inKey = proto.getInKey().toByteArray();
2752      }
2753      byte[] inIv = null;
2754      if (proto.getInIv() != null) {
2755        inIv = proto.getInIv().toByteArray();
2756      }
2757      byte[] outKey = null;
2758      if (proto.getOutKey() != null) {
2759        outKey = proto.getOutKey().toByteArray();
2760      }
2761      byte[] outIv = null;
2762      if (proto.getOutIv() != null) {
2763        outIv = proto.getOutIv().toByteArray();
2764      }
2765      return new CipherOption(suite, inKey, inIv, outKey, outIv);
2766    }
2767    return null;
2768  }
2769  
2770  public static List<CipherOptionProto> convertCipherOptions(
2771      List<CipherOption> options) {
2772    if (options != null) {
2773      List<CipherOptionProto> protos = 
2774          Lists.newArrayListWithCapacity(options.size());
2775      for (CipherOption option : options) {
2776        protos.add(convert(option));
2777      }
2778      return protos;
2779    }
2780    return null;
2781  }
2782  
2783  public static List<CipherOption> convertCipherOptionProtos(
2784      List<CipherOptionProto> protos) {
2785    if (protos != null) {
2786      List<CipherOption> options = 
2787          Lists.newArrayListWithCapacity(protos.size());
2788      for (CipherOptionProto proto : protos) {
2789        options.add(convert(proto));
2790      }
2791      return options;
2792    }
2793    return null;
2794  }
2795
2796  public static CipherSuiteProto convert(CipherSuite suite) {
2797    switch (suite) {
2798    case UNKNOWN:
2799      return CipherSuiteProto.UNKNOWN;
2800    case AES_CTR_NOPADDING:
2801      return CipherSuiteProto.AES_CTR_NOPADDING;
2802    default:
2803      return null;
2804    }
2805  }
2806
2807  public static CipherSuite convert(CipherSuiteProto proto) {
2808    switch (proto) {
2809    case AES_CTR_NOPADDING:
2810      return CipherSuite.AES_CTR_NOPADDING;
2811    default:
2812      // Set to UNKNOWN and stash the unknown enum value
2813      CipherSuite suite = CipherSuite.UNKNOWN;
2814      suite.setUnknownValue(proto.getNumber());
2815      return suite;
2816    }
2817  }
2818
2819  public static List<CryptoProtocolVersionProto> convert(
2820      CryptoProtocolVersion[] versions) {
2821    List<CryptoProtocolVersionProto> protos =
2822        Lists.newArrayListWithCapacity(versions.length);
2823    for (CryptoProtocolVersion v: versions) {
2824      protos.add(convert(v));
2825    }
2826    return protos;
2827  }
2828
2829  public static CryptoProtocolVersion[] convertCryptoProtocolVersions(
2830      List<CryptoProtocolVersionProto> protos) {
2831    List<CryptoProtocolVersion> versions =
2832        Lists.newArrayListWithCapacity(protos.size());
2833    for (CryptoProtocolVersionProto p: protos) {
2834      versions.add(convert(p));
2835    }
2836    return versions.toArray(new CryptoProtocolVersion[] {});
2837  }
2838
2839  public static CryptoProtocolVersion convert(CryptoProtocolVersionProto
2840      proto) {
2841    switch(proto) {
2842    case ENCRYPTION_ZONES:
2843      return CryptoProtocolVersion.ENCRYPTION_ZONES;
2844    default:
2845      // Set to UNKNOWN and stash the unknown enum value
2846      CryptoProtocolVersion version = CryptoProtocolVersion.UNKNOWN;
2847      version.setUnknownValue(proto.getNumber());
2848      return version;
2849    }
2850  }
2851
2852  public static CryptoProtocolVersionProto convert(CryptoProtocolVersion
2853      version) {
2854    switch(version) {
2855    case UNKNOWN:
2856      return CryptoProtocolVersionProto.UNKNOWN_PROTOCOL_VERSION;
2857    case ENCRYPTION_ZONES:
2858      return CryptoProtocolVersionProto.ENCRYPTION_ZONES;
2859    default:
2860      return null;
2861    }
2862  }
2863
2864  public static HdfsProtos.FileEncryptionInfoProto convert(
2865      FileEncryptionInfo info) {
2866    if (info == null) {
2867      return null;
2868    }
2869    return HdfsProtos.FileEncryptionInfoProto.newBuilder()
2870        .setSuite(convert(info.getCipherSuite()))
2871        .setCryptoProtocolVersion(convert(info.getCryptoProtocolVersion()))
2872        .setKey(getByteString(info.getEncryptedDataEncryptionKey()))
2873        .setIv(getByteString(info.getIV()))
2874        .setEzKeyVersionName(info.getEzKeyVersionName())
2875        .setKeyName(info.getKeyName())
2876        .build();
2877  }
2878
2879  public static HdfsProtos.PerFileEncryptionInfoProto convertPerFileEncInfo(
2880      FileEncryptionInfo info) {
2881    if (info == null) {
2882      return null;
2883    }
2884    return HdfsProtos.PerFileEncryptionInfoProto.newBuilder()
2885        .setKey(getByteString(info.getEncryptedDataEncryptionKey()))
2886        .setIv(getByteString(info.getIV()))
2887        .setEzKeyVersionName(info.getEzKeyVersionName())
2888        .build();
2889  }
2890
2891  public static HdfsProtos.ZoneEncryptionInfoProto convert(
2892      CipherSuite suite, CryptoProtocolVersion version, String keyName) {
2893    if (suite == null || version == null || keyName == null) {
2894      return null;
2895    }
2896    return HdfsProtos.ZoneEncryptionInfoProto.newBuilder()
2897        .setSuite(convert(suite))
2898        .setCryptoProtocolVersion(convert(version))
2899        .setKeyName(keyName)
2900        .build();
2901  }
2902
2903  public static FileEncryptionInfo convert(
2904      HdfsProtos.FileEncryptionInfoProto proto) {
2905    if (proto == null) {
2906      return null;
2907    }
2908    CipherSuite suite = convert(proto.getSuite());
2909    CryptoProtocolVersion version = convert(proto.getCryptoProtocolVersion());
2910    byte[] key = proto.getKey().toByteArray();
2911    byte[] iv = proto.getIv().toByteArray();
2912    String ezKeyVersionName = proto.getEzKeyVersionName();
2913    String keyName = proto.getKeyName();
2914    return new FileEncryptionInfo(suite, version, key, iv, keyName,
2915        ezKeyVersionName);
2916  }
2917
2918  public static FileEncryptionInfo convert(
2919      HdfsProtos.PerFileEncryptionInfoProto fileProto,
2920      CipherSuite suite, CryptoProtocolVersion version, String keyName) {
2921    if (fileProto == null || suite == null || version == null ||
2922        keyName == null) {
2923      return null;
2924    }
2925    byte[] key = fileProto.getKey().toByteArray();
2926    byte[] iv = fileProto.getIv().toByteArray();
2927    String ezKeyVersionName = fileProto.getEzKeyVersionName();
2928    return new FileEncryptionInfo(suite, version, key, iv, keyName,
2929        ezKeyVersionName);
2930  }
2931
2932  public static BlockReportContext convert(BlockReportContextProto proto) {
2933    return new BlockReportContext(proto.getTotalRpcs(),
2934        proto.getCurRpc(), proto.getId());
2935  }
2936
2937  public static BlockReportContextProto convert(BlockReportContext context) {
2938    return BlockReportContextProto.newBuilder().
2939        setTotalRpcs(context.getTotalRpcs()).
2940        setCurRpc(context.getCurRpc()).
2941        setId(context.getReportId()).
2942        build();
2943  }
2944}