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;
021
022import java.io.EOFException;
023import java.io.IOException;
024import java.io.InputStream;
025import java.util.ArrayList;
026import java.util.Arrays;
027import java.util.EnumSet;
028import java.util.List;
029
030import org.apache.hadoop.fs.CacheFlag;
031import org.apache.hadoop.fs.ContentSummary;
032import org.apache.hadoop.fs.CreateFlag;
033import org.apache.hadoop.fs.FsServerDefaults;
034import org.apache.hadoop.fs.Path;
035import org.apache.hadoop.fs.XAttr;
036import org.apache.hadoop.fs.XAttrSetFlag;
037import org.apache.hadoop.fs.permission.AclEntry;
038import org.apache.hadoop.fs.permission.AclEntryScope;
039import org.apache.hadoop.fs.permission.AclEntryType;
040import org.apache.hadoop.fs.permission.AclStatus;
041import org.apache.hadoop.fs.permission.FsAction;
042import org.apache.hadoop.fs.permission.FsPermission;
043import org.apache.hadoop.ha.HAServiceProtocol.HAServiceState;
044import org.apache.hadoop.hdfs.DFSUtil;
045import org.apache.hadoop.hdfs.StorageType;
046import org.apache.hadoop.hdfs.protocol.Block;
047import org.apache.hadoop.hdfs.protocol.CacheDirectiveEntry;
048import org.apache.hadoop.hdfs.protocol.CacheDirectiveInfo;
049import org.apache.hadoop.hdfs.protocol.CacheDirectiveStats;
050import org.apache.hadoop.hdfs.protocol.CachePoolEntry;
051import org.apache.hadoop.hdfs.protocol.CachePoolInfo;
052import org.apache.hadoop.hdfs.protocol.CachePoolStats;
053import org.apache.hadoop.hdfs.protocol.ClientProtocol;
054import org.apache.hadoop.hdfs.protocol.CorruptFileBlocks;
055import org.apache.hadoop.hdfs.protocol.DatanodeID;
056import org.apache.hadoop.hdfs.protocol.DatanodeInfo;
057import org.apache.hadoop.hdfs.protocol.DatanodeInfo.AdminStates;
058import org.apache.hadoop.hdfs.protocol.DatanodeLocalInfo;
059import org.apache.hadoop.hdfs.protocol.DirectoryListing;
060import org.apache.hadoop.hdfs.protocol.ExtendedBlock;
061import org.apache.hadoop.hdfs.protocol.FsAclPermission;
062import org.apache.hadoop.hdfs.protocol.HdfsConstants.DatanodeReportType;
063import org.apache.hadoop.hdfs.protocol.HdfsConstants.RollingUpgradeAction;
064import org.apache.hadoop.hdfs.protocol.HdfsConstants.SafeModeAction;
065import org.apache.hadoop.hdfs.protocol.HdfsFileStatus;
066import org.apache.hadoop.hdfs.protocol.HdfsLocatedFileStatus;
067import org.apache.hadoop.hdfs.protocol.LocatedBlock;
068import org.apache.hadoop.hdfs.protocol.LocatedBlocks;
069import org.apache.hadoop.hdfs.protocol.RollingUpgradeInfo;
070import org.apache.hadoop.hdfs.protocol.RollingUpgradeStatus;
071import org.apache.hadoop.hdfs.protocol.SnapshotDiffReport;
072import org.apache.hadoop.hdfs.protocol.SnapshotDiffReport.DiffReportEntry;
073import org.apache.hadoop.hdfs.protocol.SnapshotDiffReport.DiffType;
074import org.apache.hadoop.hdfs.protocol.SnapshottableDirectoryStatus;
075import org.apache.hadoop.hdfs.protocol.proto.AclProtos.AclEntryProto;
076import org.apache.hadoop.hdfs.protocol.proto.AclProtos.AclEntryProto.AclEntryScopeProto;
077import org.apache.hadoop.hdfs.protocol.proto.AclProtos.AclEntryProto.AclEntryTypeProto;
078import org.apache.hadoop.hdfs.protocol.proto.AclProtos.AclEntryProto.FsActionProto;
079import org.apache.hadoop.hdfs.protocol.proto.AclProtos.AclStatusProto;
080import org.apache.hadoop.hdfs.protocol.proto.AclProtos.GetAclStatusResponseProto;
081import org.apache.hadoop.hdfs.protocol.proto.ClientNamenodeProtocolProtos;
082import org.apache.hadoop.hdfs.protocol.proto.ClientNamenodeProtocolProtos.CacheDirectiveEntryProto;
083import org.apache.hadoop.hdfs.protocol.proto.ClientNamenodeProtocolProtos.CacheDirectiveInfoExpirationProto;
084import org.apache.hadoop.hdfs.protocol.proto.ClientNamenodeProtocolProtos.CacheDirectiveInfoProto;
085import org.apache.hadoop.hdfs.protocol.proto.ClientNamenodeProtocolProtos.CacheDirectiveStatsProto;
086import org.apache.hadoop.hdfs.protocol.proto.ClientNamenodeProtocolProtos.CacheFlagProto;
087import org.apache.hadoop.hdfs.protocol.proto.ClientNamenodeProtocolProtos.CachePoolEntryProto;
088import org.apache.hadoop.hdfs.protocol.proto.ClientNamenodeProtocolProtos.CachePoolInfoProto;
089import org.apache.hadoop.hdfs.protocol.proto.ClientNamenodeProtocolProtos.CachePoolStatsProto;
090import org.apache.hadoop.hdfs.protocol.proto.ClientNamenodeProtocolProtos.CreateFlagProto;
091import org.apache.hadoop.hdfs.protocol.proto.ClientNamenodeProtocolProtos.DatanodeReportTypeProto;
092import org.apache.hadoop.hdfs.protocol.proto.ClientNamenodeProtocolProtos.GetFsStatsResponseProto;
093import org.apache.hadoop.hdfs.protocol.proto.ClientNamenodeProtocolProtos.RollingUpgradeActionProto;
094import org.apache.hadoop.hdfs.protocol.proto.ClientNamenodeProtocolProtos.RollingUpgradeInfoProto;
095import org.apache.hadoop.hdfs.protocol.proto.ClientNamenodeProtocolProtos.SafeModeActionProto;
096import org.apache.hadoop.hdfs.protocol.proto.DataTransferProtos.ShortCircuitShmIdProto;
097import org.apache.hadoop.hdfs.protocol.proto.DataTransferProtos.ShortCircuitShmSlotProto;
098import org.apache.hadoop.hdfs.protocol.proto.DatanodeProtocolProtos.BalancerBandwidthCommandProto;
099import org.apache.hadoop.hdfs.protocol.proto.DatanodeProtocolProtos.BlockCommandProto;
100import org.apache.hadoop.hdfs.protocol.proto.DatanodeProtocolProtos.BlockIdCommandProto;
101import org.apache.hadoop.hdfs.protocol.proto.DatanodeProtocolProtos.BlockRecoveryCommandProto;
102import org.apache.hadoop.hdfs.protocol.proto.DatanodeProtocolProtos.DatanodeCommandProto;
103import org.apache.hadoop.hdfs.protocol.proto.DatanodeProtocolProtos.DatanodeRegistrationProto;
104import org.apache.hadoop.hdfs.protocol.proto.DatanodeProtocolProtos.DatanodeStorageProto;
105import org.apache.hadoop.hdfs.protocol.proto.DatanodeProtocolProtos.DatanodeStorageProto.StorageState;
106import org.apache.hadoop.hdfs.protocol.proto.DatanodeProtocolProtos.FinalizeCommandProto;
107import org.apache.hadoop.hdfs.protocol.proto.DatanodeProtocolProtos.KeyUpdateCommandProto;
108import org.apache.hadoop.hdfs.protocol.proto.DatanodeProtocolProtos.NNHAStatusHeartbeatProto;
109import org.apache.hadoop.hdfs.protocol.proto.DatanodeProtocolProtos.ReceivedDeletedBlockInfoProto;
110import org.apache.hadoop.hdfs.protocol.proto.DatanodeProtocolProtos.RegisterCommandProto;
111import org.apache.hadoop.hdfs.protocol.proto.DatanodeProtocolProtos.StorageReportProto;
112import org.apache.hadoop.hdfs.protocol.proto.HdfsProtos;
113import org.apache.hadoop.hdfs.protocol.proto.HdfsProtos.BlockKeyProto;
114import org.apache.hadoop.hdfs.protocol.proto.HdfsProtos.BlockProto;
115import org.apache.hadoop.hdfs.protocol.proto.HdfsProtos.BlockWithLocationsProto;
116import org.apache.hadoop.hdfs.protocol.proto.HdfsProtos.BlocksWithLocationsProto;
117import org.apache.hadoop.hdfs.protocol.proto.HdfsProtos.CheckpointCommandProto;
118import org.apache.hadoop.hdfs.protocol.proto.HdfsProtos.CheckpointSignatureProto;
119import org.apache.hadoop.hdfs.protocol.proto.HdfsProtos.ContentSummaryProto;
120import org.apache.hadoop.hdfs.protocol.proto.HdfsProtos.CorruptFileBlocksProto;
121import org.apache.hadoop.hdfs.protocol.proto.HdfsProtos.DataEncryptionKeyProto;
122import org.apache.hadoop.hdfs.protocol.proto.HdfsProtos.DatanodeIDProto;
123import org.apache.hadoop.hdfs.protocol.proto.HdfsProtos.DatanodeInfoProto;
124import org.apache.hadoop.hdfs.protocol.proto.HdfsProtos.DatanodeInfoProto.AdminState;
125import org.apache.hadoop.hdfs.protocol.proto.HdfsProtos.DatanodeInfosProto;
126import org.apache.hadoop.hdfs.protocol.proto.HdfsProtos.DatanodeLocalInfoProto;
127import org.apache.hadoop.hdfs.protocol.proto.HdfsProtos.DirectoryListingProto;
128import org.apache.hadoop.hdfs.protocol.proto.HdfsProtos.ExportedBlockKeysProto;
129import org.apache.hadoop.hdfs.protocol.proto.HdfsProtos.ExtendedBlockProto;
130import org.apache.hadoop.hdfs.protocol.proto.HdfsProtos.FsPermissionProto;
131import org.apache.hadoop.hdfs.protocol.proto.HdfsProtos.FsServerDefaultsProto;
132import org.apache.hadoop.hdfs.protocol.proto.HdfsProtos.HdfsFileStatusProto;
133import org.apache.hadoop.hdfs.protocol.proto.HdfsProtos.HdfsFileStatusProto.FileType;
134import org.apache.hadoop.hdfs.protocol.proto.HdfsProtos.LocatedBlockProto;
135import org.apache.hadoop.hdfs.protocol.proto.HdfsProtos.LocatedBlockProto.Builder;
136import org.apache.hadoop.hdfs.protocol.proto.HdfsProtos.LocatedBlocksProto;
137import org.apache.hadoop.hdfs.protocol.proto.HdfsProtos.NamenodeCommandProto;
138import org.apache.hadoop.hdfs.protocol.proto.HdfsProtos.NamenodeRegistrationProto;
139import org.apache.hadoop.hdfs.protocol.proto.HdfsProtos.NamenodeRegistrationProto.NamenodeRoleProto;
140import org.apache.hadoop.hdfs.protocol.proto.HdfsProtos.NamespaceInfoProto;
141import org.apache.hadoop.hdfs.protocol.proto.HdfsProtos.RecoveringBlockProto;
142import org.apache.hadoop.hdfs.protocol.proto.HdfsProtos.RemoteEditLogManifestProto;
143import org.apache.hadoop.hdfs.protocol.proto.HdfsProtos.RemoteEditLogProto;
144import org.apache.hadoop.hdfs.protocol.proto.HdfsProtos.ReplicaStateProto;
145import org.apache.hadoop.hdfs.protocol.proto.HdfsProtos.RollingUpgradeStatusProto;
146import org.apache.hadoop.hdfs.protocol.proto.HdfsProtos.SnapshotDiffReportEntryProto;
147import org.apache.hadoop.hdfs.protocol.proto.HdfsProtos.SnapshotDiffReportProto;
148import org.apache.hadoop.hdfs.protocol.proto.HdfsProtos.SnapshottableDirectoryListingProto;
149import org.apache.hadoop.hdfs.protocol.proto.HdfsProtos.SnapshottableDirectoryStatusProto;
150import org.apache.hadoop.hdfs.protocol.proto.HdfsProtos.StorageInfoProto;
151import org.apache.hadoop.hdfs.protocol.proto.HdfsProtos.StorageTypeProto;
152import org.apache.hadoop.hdfs.protocol.proto.HdfsProtos.StorageUuidsProto;
153import org.apache.hadoop.hdfs.protocol.proto.JournalProtocolProtos.JournalInfoProto;
154import org.apache.hadoop.hdfs.protocol.proto.XAttrProtos.GetXAttrsResponseProto;
155import org.apache.hadoop.hdfs.protocol.proto.XAttrProtos.ListXAttrsResponseProto;
156import org.apache.hadoop.hdfs.protocol.proto.XAttrProtos.XAttrProto;
157import org.apache.hadoop.hdfs.protocol.proto.XAttrProtos.XAttrProto.XAttrNamespaceProto;
158import org.apache.hadoop.hdfs.protocol.proto.XAttrProtos.XAttrSetFlagProto;
159import org.apache.hadoop.hdfs.security.token.block.BlockKey;
160import org.apache.hadoop.hdfs.security.token.block.BlockTokenIdentifier;
161import org.apache.hadoop.hdfs.security.token.block.DataEncryptionKey;
162import org.apache.hadoop.hdfs.security.token.block.ExportedBlockKeys;
163import org.apache.hadoop.hdfs.security.token.delegation.DelegationTokenIdentifier;
164import org.apache.hadoop.hdfs.server.common.HdfsServerConstants.NamenodeRole;
165import org.apache.hadoop.hdfs.server.common.HdfsServerConstants.NodeType;
166import org.apache.hadoop.hdfs.server.common.HdfsServerConstants.ReplicaState;
167import org.apache.hadoop.hdfs.server.common.StorageInfo;
168import org.apache.hadoop.hdfs.server.namenode.CheckpointSignature;
169import org.apache.hadoop.hdfs.server.namenode.INodeId;
170import org.apache.hadoop.hdfs.server.protocol.BalancerBandwidthCommand;
171import org.apache.hadoop.hdfs.server.protocol.BlockCommand;
172import org.apache.hadoop.hdfs.server.protocol.BlockIdCommand;
173import org.apache.hadoop.hdfs.server.protocol.BlockRecoveryCommand;
174import org.apache.hadoop.hdfs.server.protocol.BlockRecoveryCommand.RecoveringBlock;
175import org.apache.hadoop.hdfs.server.protocol.BlocksWithLocations;
176import org.apache.hadoop.hdfs.server.protocol.BlocksWithLocations.BlockWithLocations;
177import org.apache.hadoop.hdfs.server.protocol.CheckpointCommand;
178import org.apache.hadoop.hdfs.server.protocol.DatanodeCommand;
179import org.apache.hadoop.hdfs.server.protocol.DatanodeProtocol;
180import org.apache.hadoop.hdfs.server.protocol.DatanodeRegistration;
181import org.apache.hadoop.hdfs.server.protocol.DatanodeStorage;
182import org.apache.hadoop.hdfs.server.protocol.DatanodeStorage.State;
183import org.apache.hadoop.hdfs.server.protocol.FinalizeCommand;
184import org.apache.hadoop.hdfs.server.protocol.JournalInfo;
185import org.apache.hadoop.hdfs.server.protocol.KeyUpdateCommand;
186import org.apache.hadoop.hdfs.server.protocol.NNHAStatusHeartbeat;
187import org.apache.hadoop.hdfs.server.protocol.NamenodeCommand;
188import org.apache.hadoop.hdfs.server.protocol.NamenodeRegistration;
189import org.apache.hadoop.hdfs.server.protocol.NamespaceInfo;
190import org.apache.hadoop.hdfs.server.protocol.ReceivedDeletedBlockInfo;
191import org.apache.hadoop.hdfs.server.protocol.ReceivedDeletedBlockInfo.BlockStatus;
192import org.apache.hadoop.hdfs.server.protocol.RegisterCommand;
193import org.apache.hadoop.hdfs.server.protocol.RemoteEditLog;
194import org.apache.hadoop.hdfs.server.protocol.RemoteEditLogManifest;
195import org.apache.hadoop.hdfs.server.protocol.StorageReport;
196import org.apache.hadoop.hdfs.shortcircuit.ShortCircuitShm.ShmId;
197import org.apache.hadoop.hdfs.shortcircuit.ShortCircuitShm.SlotId;
198import org.apache.hadoop.hdfs.util.ExactSizeInputStream;
199import org.apache.hadoop.io.EnumSetWritable;
200import org.apache.hadoop.io.Text;
201import org.apache.hadoop.security.proto.SecurityProtos.TokenProto;
202import org.apache.hadoop.security.token.Token;
203import org.apache.hadoop.util.DataChecksum;
204
205import com.google.common.base.Preconditions;
206import com.google.common.collect.Lists;
207import com.google.common.primitives.Shorts;
208import com.google.protobuf.ByteString;
209import com.google.protobuf.CodedInputStream;
210
211/**
212 * Utilities for converting protobuf classes to and from implementation classes
213 * and other helper utilities to help in dealing with protobuf.
214 * 
215 * Note that when converting from an internal type to protobuf type, the
216 * converter never return null for protobuf type. The check for internal type
217 * being null must be done before calling the convert() method.
218 */
219public class PBHelper {
220  private static final RegisterCommandProto REG_CMD_PROTO = 
221      RegisterCommandProto.newBuilder().build();
222  private static final RegisterCommand REG_CMD = new RegisterCommand();
223
224  private static final AclEntryScope[] ACL_ENTRY_SCOPE_VALUES =
225      AclEntryScope.values();
226  private static final AclEntryType[] ACL_ENTRY_TYPE_VALUES =
227      AclEntryType.values();
228  private static final FsAction[] FSACTION_VALUES =
229      FsAction.values();
230  private static final XAttr.NameSpace[] XATTR_NAMESPACE_VALUES = 
231      XAttr.NameSpace.values();
232
233  private PBHelper() {
234    /** Hidden constructor */
235  }
236
237  public static ByteString getByteString(byte[] bytes) {
238    return ByteString.copyFrom(bytes);
239  }
240
241  private static <T extends Enum<T>, U extends Enum<U>> U castEnum(T from, U[] to) {
242    return to[from.ordinal()];
243  }
244
245  public static NamenodeRole convert(NamenodeRoleProto role) {
246    switch (role) {
247    case NAMENODE:
248      return NamenodeRole.NAMENODE;
249    case BACKUP:
250      return NamenodeRole.BACKUP;
251    case CHECKPOINT:
252      return NamenodeRole.CHECKPOINT;
253    }
254    return null;
255  }
256
257  public static NamenodeRoleProto convert(NamenodeRole role) {
258    switch (role) {
259    case NAMENODE:
260      return NamenodeRoleProto.NAMENODE;
261    case BACKUP:
262      return NamenodeRoleProto.BACKUP;
263    case CHECKPOINT:
264      return NamenodeRoleProto.CHECKPOINT;
265    }
266    return null;
267  }
268
269  public static StorageInfoProto convert(StorageInfo info) {
270    return StorageInfoProto.newBuilder().setClusterID(info.getClusterID())
271        .setCTime(info.getCTime()).setLayoutVersion(info.getLayoutVersion())
272        .setNamespceID(info.getNamespaceID()).build();
273  }
274
275  public static StorageInfo convert(StorageInfoProto info, NodeType type) {
276    return new StorageInfo(info.getLayoutVersion(), info.getNamespceID(),
277        info.getClusterID(), info.getCTime(), type);
278  }
279
280  public static NamenodeRegistrationProto convert(NamenodeRegistration reg) {
281    return NamenodeRegistrationProto.newBuilder()
282        .setHttpAddress(reg.getHttpAddress()).setRole(convert(reg.getRole()))
283        .setRpcAddress(reg.getAddress())
284        .setStorageInfo(convert((StorageInfo) reg)).build();
285  }
286
287  public static NamenodeRegistration convert(NamenodeRegistrationProto reg) {
288    StorageInfo si = convert(reg.getStorageInfo(), NodeType.NAME_NODE);
289    return new NamenodeRegistration(reg.getRpcAddress(), reg.getHttpAddress(),
290        si, convert(reg.getRole()));
291  }
292
293  // DatanodeId
294  public static DatanodeID convert(DatanodeIDProto dn) {
295    return new DatanodeID(dn.getIpAddr(), dn.getHostName(), dn.getDatanodeUuid(),
296        dn.getXferPort(), dn.getInfoPort(), dn.hasInfoSecurePort() ? dn
297        .getInfoSecurePort() : 0, dn.getIpcPort());
298  }
299
300  public static DatanodeIDProto convert(DatanodeID dn) {
301    // For wire compatibility with older versions we transmit the StorageID
302    // which is the same as the DatanodeUuid. Since StorageID is a required
303    // field we pass the empty string if the DatanodeUuid is not yet known.
304    return DatanodeIDProto.newBuilder()
305        .setIpAddr(dn.getIpAddr())
306        .setHostName(dn.getHostName())
307        .setXferPort(dn.getXferPort())
308        .setDatanodeUuid(dn.getDatanodeUuid() != null ? dn.getDatanodeUuid() : "")
309        .setInfoPort(dn.getInfoPort())
310        .setInfoSecurePort(dn.getInfoSecurePort())
311        .setIpcPort(dn.getIpcPort()).build();
312  }
313
314  // Arrays of DatanodeId
315  public static DatanodeIDProto[] convert(DatanodeID[] did) {
316    if (did == null)
317      return null;
318    final int len = did.length;
319    DatanodeIDProto[] result = new DatanodeIDProto[len];
320    for (int i = 0; i < len; ++i) {
321      result[i] = convert(did[i]);
322    }
323    return result;
324  }
325  
326  public static DatanodeID[] convert(DatanodeIDProto[] did) {
327    if (did == null) return null;
328    final int len = did.length;
329    DatanodeID[] result = new DatanodeID[len];
330    for (int i = 0; i < len; ++i) {
331      result[i] = convert(did[i]);
332    }
333    return result;
334  }
335  
336  // Block
337  public static BlockProto convert(Block b) {
338    return BlockProto.newBuilder().setBlockId(b.getBlockId())
339        .setGenStamp(b.getGenerationStamp()).setNumBytes(b.getNumBytes())
340        .build();
341  }
342
343  public static Block convert(BlockProto b) {
344    return new Block(b.getBlockId(), b.getNumBytes(), b.getGenStamp());
345  }
346
347  public static BlockWithLocationsProto convert(BlockWithLocations blk) {
348    return BlockWithLocationsProto.newBuilder()
349        .setBlock(convert(blk.getBlock()))
350        .addAllDatanodeUuids(Arrays.asList(blk.getDatanodeUuids()))
351        .addAllStorageUuids(Arrays.asList(blk.getStorageIDs())).build();
352  }
353
354  public static BlockWithLocations convert(BlockWithLocationsProto b) {
355    final List<String> datanodeUuids = b.getDatanodeUuidsList();
356    final List<String> storageUuids = b.getStorageUuidsList();
357    return new BlockWithLocations(convert(b.getBlock()),
358        datanodeUuids.toArray(new String[datanodeUuids.size()]),
359        storageUuids.toArray(new String[storageUuids.size()]));
360  }
361
362  public static BlocksWithLocationsProto convert(BlocksWithLocations blks) {
363    BlocksWithLocationsProto.Builder builder = BlocksWithLocationsProto
364        .newBuilder();
365    for (BlockWithLocations b : blks.getBlocks()) {
366      builder.addBlocks(convert(b));
367    }
368    return builder.build();
369  }
370
371  public static BlocksWithLocations convert(BlocksWithLocationsProto blocks) {
372    List<BlockWithLocationsProto> b = blocks.getBlocksList();
373    BlockWithLocations[] ret = new BlockWithLocations[b.size()];
374    int i = 0;
375    for (BlockWithLocationsProto entry : b) {
376      ret[i++] = convert(entry);
377    }
378    return new BlocksWithLocations(ret);
379  }
380
381  public static BlockKeyProto convert(BlockKey key) {
382    byte[] encodedKey = key.getEncodedKey();
383    ByteString keyBytes = ByteString.copyFrom(encodedKey == null ? 
384        DFSUtil.EMPTY_BYTES : encodedKey);
385    return BlockKeyProto.newBuilder().setKeyId(key.getKeyId())
386        .setKeyBytes(keyBytes).setExpiryDate(key.getExpiryDate()).build();
387  }
388
389  public static BlockKey convert(BlockKeyProto k) {
390    return new BlockKey(k.getKeyId(), k.getExpiryDate(), k.getKeyBytes()
391        .toByteArray());
392  }
393
394  public static ExportedBlockKeysProto convert(ExportedBlockKeys keys) {
395    ExportedBlockKeysProto.Builder builder = ExportedBlockKeysProto
396        .newBuilder();
397    builder.setIsBlockTokenEnabled(keys.isBlockTokenEnabled())
398        .setKeyUpdateInterval(keys.getKeyUpdateInterval())
399        .setTokenLifeTime(keys.getTokenLifetime())
400        .setCurrentKey(convert(keys.getCurrentKey()));
401    for (BlockKey k : keys.getAllKeys()) {
402      builder.addAllKeys(convert(k));
403    }
404    return builder.build();
405  }
406
407  public static ExportedBlockKeys convert(ExportedBlockKeysProto keys) {
408    return new ExportedBlockKeys(keys.getIsBlockTokenEnabled(),
409        keys.getKeyUpdateInterval(), keys.getTokenLifeTime(),
410        convert(keys.getCurrentKey()), convertBlockKeys(keys.getAllKeysList()));
411  }
412
413  public static CheckpointSignatureProto convert(CheckpointSignature s) {
414    return CheckpointSignatureProto.newBuilder()
415        .setBlockPoolId(s.getBlockpoolID())
416        .setCurSegmentTxId(s.getCurSegmentTxId())
417        .setMostRecentCheckpointTxId(s.getMostRecentCheckpointTxId())
418        .setStorageInfo(PBHelper.convert((StorageInfo) s)).build();
419  }
420
421  public static CheckpointSignature convert(CheckpointSignatureProto s) {
422    StorageInfo si = PBHelper.convert(s.getStorageInfo(), NodeType.NAME_NODE);
423    return new CheckpointSignature(si, s.getBlockPoolId(),
424        s.getMostRecentCheckpointTxId(), s.getCurSegmentTxId());
425  }
426
427  public static RemoteEditLogProto convert(RemoteEditLog log) {
428    return RemoteEditLogProto.newBuilder()
429        .setStartTxId(log.getStartTxId())
430        .setEndTxId(log.getEndTxId())
431        .setIsInProgress(log.isInProgress()).build();
432  }
433
434  public static RemoteEditLog convert(RemoteEditLogProto l) {
435    return new RemoteEditLog(l.getStartTxId(), l.getEndTxId(),
436        l.getIsInProgress());
437  }
438
439  public static RemoteEditLogManifestProto convert(
440      RemoteEditLogManifest manifest) {
441    RemoteEditLogManifestProto.Builder builder = RemoteEditLogManifestProto
442        .newBuilder();
443    for (RemoteEditLog log : manifest.getLogs()) {
444      builder.addLogs(convert(log));
445    }
446    return builder.build();
447  }
448
449  public static RemoteEditLogManifest convert(
450      RemoteEditLogManifestProto manifest) {
451    List<RemoteEditLog> logs = new ArrayList<RemoteEditLog>(manifest
452        .getLogsList().size());
453    for (RemoteEditLogProto l : manifest.getLogsList()) {
454      logs.add(convert(l));
455    }
456    return new RemoteEditLogManifest(logs);
457  }
458
459  public static CheckpointCommandProto convert(CheckpointCommand cmd) {
460    return CheckpointCommandProto.newBuilder()
461        .setSignature(convert(cmd.getSignature()))
462        .setNeedToReturnImage(cmd.needToReturnImage()).build();
463  }
464
465  public static NamenodeCommandProto convert(NamenodeCommand cmd) {
466    if (cmd instanceof CheckpointCommand) {
467      return NamenodeCommandProto.newBuilder().setAction(cmd.getAction())
468          .setType(NamenodeCommandProto.Type.CheckPointCommand)
469          .setCheckpointCmd(convert((CheckpointCommand) cmd)).build();
470    }
471    return NamenodeCommandProto.newBuilder()
472        .setType(NamenodeCommandProto.Type.NamenodeCommand)
473        .setAction(cmd.getAction()).build();
474  }
475
476  public static BlockKey[] convertBlockKeys(List<BlockKeyProto> list) {
477    BlockKey[] ret = new BlockKey[list.size()];
478    int i = 0;
479    for (BlockKeyProto k : list) {
480      ret[i++] = convert(k);
481    }
482    return ret;
483  }
484
485  public static NamespaceInfo convert(NamespaceInfoProto info) {
486    StorageInfoProto storage = info.getStorageInfo();
487    return new NamespaceInfo(storage.getNamespceID(), storage.getClusterID(),
488        info.getBlockPoolID(), storage.getCTime(), info.getBuildVersion(),
489        info.getSoftwareVersion());
490  }
491
492  public static NamenodeCommand convert(NamenodeCommandProto cmd) {
493    if (cmd == null) return null;
494    switch (cmd.getType()) {
495    case CheckPointCommand:
496      CheckpointCommandProto chkPt = cmd.getCheckpointCmd();
497      return new CheckpointCommand(PBHelper.convert(chkPt.getSignature()),
498          chkPt.getNeedToReturnImage());
499    default:
500      return new NamenodeCommand(cmd.getAction());
501    }
502  }
503  
504  public static ExtendedBlock convert(ExtendedBlockProto eb) {
505    if (eb == null) return null;
506    return new ExtendedBlock( eb.getPoolId(),  eb.getBlockId(),   eb.getNumBytes(),
507       eb.getGenerationStamp());
508  }
509  
510  public static ExtendedBlockProto convert(final ExtendedBlock b) {
511    if (b == null) return null;
512   return ExtendedBlockProto.newBuilder().
513      setPoolId(b.getBlockPoolId()).
514      setBlockId(b.getBlockId()).
515      setNumBytes(b.getNumBytes()).
516      setGenerationStamp(b.getGenerationStamp()).
517      build();
518  }
519  
520  public static RecoveringBlockProto convert(RecoveringBlock b) {
521    if (b == null) {
522      return null;
523    }
524    LocatedBlockProto lb = PBHelper.convert((LocatedBlock)b);
525    return RecoveringBlockProto.newBuilder().setBlock(lb)
526        .setNewGenStamp(b.getNewGenerationStamp()).build();
527  }
528
529  public static RecoveringBlock convert(RecoveringBlockProto b) {
530    ExtendedBlock block = convert(b.getBlock().getB());
531    DatanodeInfo[] locs = convert(b.getBlock().getLocsList());
532    return new RecoveringBlock(block, locs, b.getNewGenStamp());
533  }
534  
535  public static DatanodeInfoProto.AdminState convert(
536      final DatanodeInfo.AdminStates inAs) {
537    switch (inAs) {
538    case NORMAL: return  DatanodeInfoProto.AdminState.NORMAL;
539    case DECOMMISSION_INPROGRESS: 
540        return DatanodeInfoProto.AdminState.DECOMMISSION_INPROGRESS;
541    case DECOMMISSIONED: return DatanodeInfoProto.AdminState.DECOMMISSIONED;
542    default: return DatanodeInfoProto.AdminState.NORMAL;
543    }
544  }
545  
546  static public DatanodeInfo convert(DatanodeInfoProto di) {
547    if (di == null) return null;
548    return new DatanodeInfo(
549        PBHelper.convert(di.getId()),
550        di.hasLocation() ? di.getLocation() : null , 
551        di.getCapacity(),  di.getDfsUsed(),  di.getRemaining(),
552        di.getBlockPoolUsed(), di.getCacheCapacity(), di.getCacheUsed(),
553        di.getLastUpdate(), di.getXceiverCount(),
554        PBHelper.convert(di.getAdminState())); 
555  }
556  
557  static public DatanodeInfoProto convertDatanodeInfo(DatanodeInfo di) {
558    if (di == null) return null;
559    return convert(di);
560  }
561  
562  
563  static public DatanodeInfo[] convert(DatanodeInfoProto di[]) {
564    if (di == null) return null;
565    DatanodeInfo[] result = new DatanodeInfo[di.length];
566    for (int i = 0; i < di.length; i++) {
567      result[i] = convert(di[i]);
568    }    
569    return result;
570  }
571
572  public static List<? extends HdfsProtos.DatanodeInfoProto> convert(
573      DatanodeInfo[] dnInfos) {
574    return convert(dnInfos, 0);
575  }
576  
577  /**
578   * Copy from {@code dnInfos} to a target of list of same size starting at
579   * {@code startIdx}.
580   */
581  public static List<? extends HdfsProtos.DatanodeInfoProto> convert(
582      DatanodeInfo[] dnInfos, int startIdx) {
583    if (dnInfos == null)
584      return null;
585    ArrayList<HdfsProtos.DatanodeInfoProto> protos = Lists
586        .newArrayListWithCapacity(dnInfos.length);
587    for (int i = startIdx; i < dnInfos.length; i++) {
588      protos.add(convert(dnInfos[i]));
589    }
590    return protos;
591  }
592
593  public static DatanodeInfo[] convert(List<DatanodeInfoProto> list) {
594    DatanodeInfo[] info = new DatanodeInfo[list.size()];
595    for (int i = 0; i < info.length; i++) {
596      info[i] = convert(list.get(i));
597    }
598    return info;
599  }
600  
601  public static DatanodeInfoProto convert(DatanodeInfo info) {
602    DatanodeInfoProto.Builder builder = DatanodeInfoProto.newBuilder();
603    if (info.getNetworkLocation() != null) {
604      builder.setLocation(info.getNetworkLocation());
605    }
606    builder
607        .setId(PBHelper.convert((DatanodeID)info))
608        .setCapacity(info.getCapacity())
609        .setDfsUsed(info.getDfsUsed())
610        .setRemaining(info.getRemaining())
611        .setBlockPoolUsed(info.getBlockPoolUsed())
612        .setCacheCapacity(info.getCacheCapacity())
613        .setCacheUsed(info.getCacheUsed())
614        .setLastUpdate(info.getLastUpdate())
615        .setXceiverCount(info.getXceiverCount())
616        .setAdminState(PBHelper.convert(info.getAdminState()))
617        .build();
618    return builder.build();
619  }
620
621  public static AdminStates convert(AdminState adminState) {
622    switch(adminState) {
623    case DECOMMISSION_INPROGRESS:
624      return AdminStates.DECOMMISSION_INPROGRESS;
625    case DECOMMISSIONED:
626      return AdminStates.DECOMMISSIONED;
627    case NORMAL:
628    default:
629      return AdminStates.NORMAL;
630    }
631  }
632  
633  public static LocatedBlockProto convert(LocatedBlock b) {
634    if (b == null) return null;
635    Builder builder = LocatedBlockProto.newBuilder();
636    DatanodeInfo[] locs = b.getLocations();
637    List<DatanodeInfo> cachedLocs =
638        Lists.newLinkedList(Arrays.asList(b.getCachedLocations()));
639    for (int i = 0; i < locs.length; i++) {
640      DatanodeInfo loc = locs[i];
641      builder.addLocs(i, PBHelper.convert(loc));
642      boolean locIsCached = cachedLocs.contains(loc);
643      builder.addIsCached(locIsCached);
644      if (locIsCached) {
645        cachedLocs.remove(loc);
646      }
647    }
648    Preconditions.checkArgument(cachedLocs.size() == 0,
649        "Found additional cached replica locations that are not in the set of"
650        + " storage-backed locations!");
651
652    StorageType[] storageTypes = b.getStorageTypes();
653    if (storageTypes != null) {
654      for (int i = 0; i < storageTypes.length; ++i) {
655        builder.addStorageTypes(PBHelper.convertStorageType(storageTypes[i]));
656      }
657    }
658    final String[] storageIDs = b.getStorageIDs();
659    if (storageIDs != null) {
660      builder.addAllStorageIDs(Arrays.asList(storageIDs));
661    }
662
663    return builder.setB(PBHelper.convert(b.getBlock()))
664        .setBlockToken(PBHelper.convert(b.getBlockToken()))
665        .setCorrupt(b.isCorrupt()).setOffset(b.getStartOffset()).build();
666  }
667  
668  public static LocatedBlock convert(LocatedBlockProto proto) {
669    if (proto == null) return null;
670    List<DatanodeInfoProto> locs = proto.getLocsList();
671    DatanodeInfo[] targets = new DatanodeInfo[locs.size()];
672    for (int i = 0; i < locs.size(); i++) {
673      targets[i] = PBHelper.convert(locs.get(i));
674    }
675
676    final int storageTypesCount = proto.getStorageTypesCount();
677    final StorageType[] storageTypes;
678    if (storageTypesCount == 0) {
679      storageTypes = null;
680    } else {
681      Preconditions.checkState(storageTypesCount == locs.size());
682      storageTypes = convertStorageTypeProtos(proto.getStorageTypesList());
683    }
684
685    final int storageIDsCount = proto.getStorageIDsCount();
686    final String[] storageIDs;
687    if (storageIDsCount == 0) {
688      storageIDs = null;
689    } else {
690      Preconditions.checkState(storageIDsCount == locs.size());
691      storageIDs = proto.getStorageIDsList().toArray(new String[storageIDsCount]);
692    }
693
694    // Set values from the isCached list, re-using references from loc
695    List<DatanodeInfo> cachedLocs = new ArrayList<DatanodeInfo>(locs.size());
696    List<Boolean> isCachedList = proto.getIsCachedList();
697    for (int i=0; i<isCachedList.size(); i++) {
698      if (isCachedList.get(i)) {
699        cachedLocs.add(targets[i]);
700      }
701    }
702
703    LocatedBlock lb = new LocatedBlock(PBHelper.convert(proto.getB()), targets,
704        storageIDs, storageTypes, proto.getOffset(), proto.getCorrupt(),
705        cachedLocs.toArray(new DatanodeInfo[0]));
706    lb.setBlockToken(PBHelper.convert(proto.getBlockToken()));
707
708    return lb;
709  }
710
711  public static TokenProto convert(Token<?> tok) {
712    return TokenProto.newBuilder().
713              setIdentifier(ByteString.copyFrom(tok.getIdentifier())).
714              setPassword(ByteString.copyFrom(tok.getPassword())).
715              setKind(tok.getKind().toString()).
716              setService(tok.getService().toString()).build(); 
717  }
718  
719  public static Token<BlockTokenIdentifier> convert(
720      TokenProto blockToken) {
721    return new Token<BlockTokenIdentifier>(blockToken.getIdentifier()
722        .toByteArray(), blockToken.getPassword().toByteArray(), new Text(
723        blockToken.getKind()), new Text(blockToken.getService()));
724  }
725
726  
727  public static Token<DelegationTokenIdentifier> convertDelegationToken(
728      TokenProto blockToken) {
729    return new Token<DelegationTokenIdentifier>(blockToken.getIdentifier()
730        .toByteArray(), blockToken.getPassword().toByteArray(), new Text(
731        blockToken.getKind()), new Text(blockToken.getService()));
732  }
733
734  public static ReplicaState convert(ReplicaStateProto state) {
735    switch (state) {
736    case RBW:
737      return ReplicaState.RBW;
738    case RUR:
739      return ReplicaState.RUR;
740    case RWR:
741      return ReplicaState.RWR;
742    case TEMPORARY:
743      return ReplicaState.TEMPORARY;
744    case FINALIZED:
745    default:
746      return ReplicaState.FINALIZED;
747    }
748  }
749
750  public static ReplicaStateProto convert(ReplicaState state) {
751    switch (state) {
752    case RBW:
753      return ReplicaStateProto.RBW;
754    case RUR:
755      return ReplicaStateProto.RUR;
756    case RWR:
757      return ReplicaStateProto.RWR;
758    case TEMPORARY:
759      return ReplicaStateProto.TEMPORARY;
760    case FINALIZED:
761    default:
762      return ReplicaStateProto.FINALIZED;
763    }
764  }
765  
766  public static DatanodeRegistrationProto convert(
767      DatanodeRegistration registration) {
768    DatanodeRegistrationProto.Builder builder = DatanodeRegistrationProto
769        .newBuilder();
770    return builder.setDatanodeID(PBHelper.convert((DatanodeID) registration))
771        .setStorageInfo(PBHelper.convert(registration.getStorageInfo()))
772        .setKeys(PBHelper.convert(registration.getExportedKeys()))
773        .setSoftwareVersion(registration.getSoftwareVersion()).build();
774  }
775
776  public static DatanodeRegistration convert(DatanodeRegistrationProto proto) {
777    StorageInfo si = convert(proto.getStorageInfo(), NodeType.DATA_NODE);
778    return new DatanodeRegistration(PBHelper.convert(proto.getDatanodeID()),
779        si, PBHelper.convert(proto.getKeys()), proto.getSoftwareVersion());
780  }
781
782  public static DatanodeCommand convert(DatanodeCommandProto proto) {
783    switch (proto.getCmdType()) {
784    case BalancerBandwidthCommand:
785      return PBHelper.convert(proto.getBalancerCmd());
786    case BlockCommand:
787      return PBHelper.convert(proto.getBlkCmd());
788    case BlockRecoveryCommand:
789      return PBHelper.convert(proto.getRecoveryCmd());
790    case FinalizeCommand:
791      return PBHelper.convert(proto.getFinalizeCmd());
792    case KeyUpdateCommand:
793      return PBHelper.convert(proto.getKeyUpdateCmd());
794    case RegisterCommand:
795      return REG_CMD;
796    case BlockIdCommand:
797      return PBHelper.convert(proto.getBlkIdCmd());
798    default:
799      return null;
800    }
801  }
802  
803  public static BalancerBandwidthCommandProto convert(
804      BalancerBandwidthCommand bbCmd) {
805    return BalancerBandwidthCommandProto.newBuilder()
806        .setBandwidth(bbCmd.getBalancerBandwidthValue()).build();
807  }
808
809  public static KeyUpdateCommandProto convert(KeyUpdateCommand cmd) {
810    return KeyUpdateCommandProto.newBuilder()
811        .setKeys(PBHelper.convert(cmd.getExportedKeys())).build();
812  }
813
814  public static BlockRecoveryCommandProto convert(BlockRecoveryCommand cmd) {
815    BlockRecoveryCommandProto.Builder builder = BlockRecoveryCommandProto
816        .newBuilder();
817    for (RecoveringBlock b : cmd.getRecoveringBlocks()) {
818      builder.addBlocks(PBHelper.convert(b));
819    }
820    return builder.build();
821  }
822
823  public static FinalizeCommandProto convert(FinalizeCommand cmd) {
824    return FinalizeCommandProto.newBuilder()
825        .setBlockPoolId(cmd.getBlockPoolId()).build();
826  }
827
828  public static BlockCommandProto convert(BlockCommand cmd) {
829    BlockCommandProto.Builder builder = BlockCommandProto.newBuilder()
830        .setBlockPoolId(cmd.getBlockPoolId());
831    switch (cmd.getAction()) {
832    case DatanodeProtocol.DNA_TRANSFER:
833      builder.setAction(BlockCommandProto.Action.TRANSFER);
834      break;
835    case DatanodeProtocol.DNA_INVALIDATE:
836      builder.setAction(BlockCommandProto.Action.INVALIDATE);
837      break;
838    case DatanodeProtocol.DNA_SHUTDOWN:
839      builder.setAction(BlockCommandProto.Action.SHUTDOWN);
840      break;
841    default:
842      throw new AssertionError("Invalid action");
843    }
844    Block[] blocks = cmd.getBlocks();
845    for (int i = 0; i < blocks.length; i++) {
846      builder.addBlocks(PBHelper.convert(blocks[i]));
847    }
848    builder.addAllTargets(convert(cmd.getTargets()))
849           .addAllTargetStorageUuids(convert(cmd.getTargetStorageIDs()));
850    return builder.build();
851  }
852  
853  public static BlockIdCommandProto convert(BlockIdCommand cmd) {
854    BlockIdCommandProto.Builder builder = BlockIdCommandProto.newBuilder()
855        .setBlockPoolId(cmd.getBlockPoolId());
856    switch (cmd.getAction()) {
857    case DatanodeProtocol.DNA_CACHE:
858      builder.setAction(BlockIdCommandProto.Action.CACHE);
859      break;
860    case DatanodeProtocol.DNA_UNCACHE:
861      builder.setAction(BlockIdCommandProto.Action.UNCACHE);
862      break;
863    default:
864      throw new AssertionError("Invalid action");
865    }
866    long[] blockIds = cmd.getBlockIds();
867    for (int i = 0; i < blockIds.length; i++) {
868      builder.addBlockIds(blockIds[i]);
869    }
870    return builder.build();
871  }
872
873  private static List<DatanodeInfosProto> convert(DatanodeInfo[][] targets) {
874    DatanodeInfosProto[] ret = new DatanodeInfosProto[targets.length];
875    for (int i = 0; i < targets.length; i++) {
876      ret[i] = DatanodeInfosProto.newBuilder()
877          .addAllDatanodes(PBHelper.convert(targets[i])).build();
878    }
879    return Arrays.asList(ret);
880  }
881
882  private static List<StorageUuidsProto> convert(String[][] targetStorageUuids) {
883    StorageUuidsProto[] ret = new StorageUuidsProto[targetStorageUuids.length];
884    for (int i = 0; i < targetStorageUuids.length; i++) {
885      ret[i] = StorageUuidsProto.newBuilder()
886          .addAllStorageUuids(Arrays.asList(targetStorageUuids[i])).build();
887    }
888    return Arrays.asList(ret);
889  }
890
891  public static DatanodeCommandProto convert(DatanodeCommand datanodeCommand) {
892    DatanodeCommandProto.Builder builder = DatanodeCommandProto.newBuilder();
893    if (datanodeCommand == null) {
894      return builder.setCmdType(DatanodeCommandProto.Type.NullDatanodeCommand)
895          .build();
896    }
897    switch (datanodeCommand.getAction()) {
898    case DatanodeProtocol.DNA_BALANCERBANDWIDTHUPDATE:
899      builder.setCmdType(DatanodeCommandProto.Type.BalancerBandwidthCommand)
900          .setBalancerCmd(
901              PBHelper.convert((BalancerBandwidthCommand) datanodeCommand));
902      break;
903    case DatanodeProtocol.DNA_ACCESSKEYUPDATE:
904      builder
905          .setCmdType(DatanodeCommandProto.Type.KeyUpdateCommand)
906          .setKeyUpdateCmd(PBHelper.convert((KeyUpdateCommand) datanodeCommand));
907      break;
908    case DatanodeProtocol.DNA_RECOVERBLOCK:
909      builder.setCmdType(DatanodeCommandProto.Type.BlockRecoveryCommand)
910          .setRecoveryCmd(
911              PBHelper.convert((BlockRecoveryCommand) datanodeCommand));
912      break;
913    case DatanodeProtocol.DNA_FINALIZE:
914      builder.setCmdType(DatanodeCommandProto.Type.FinalizeCommand)
915          .setFinalizeCmd(PBHelper.convert((FinalizeCommand) datanodeCommand));
916      break;
917    case DatanodeProtocol.DNA_REGISTER:
918      builder.setCmdType(DatanodeCommandProto.Type.RegisterCommand)
919          .setRegisterCmd(REG_CMD_PROTO);
920      break;
921    case DatanodeProtocol.DNA_TRANSFER:
922    case DatanodeProtocol.DNA_INVALIDATE:
923    case DatanodeProtocol.DNA_SHUTDOWN:
924      builder.setCmdType(DatanodeCommandProto.Type.BlockCommand).
925        setBlkCmd(PBHelper.convert((BlockCommand) datanodeCommand));
926      break;
927    case DatanodeProtocol.DNA_CACHE:
928    case DatanodeProtocol.DNA_UNCACHE:
929      builder.setCmdType(DatanodeCommandProto.Type.BlockIdCommand).
930        setBlkIdCmd(PBHelper.convert((BlockIdCommand) datanodeCommand));
931      break;
932    case DatanodeProtocol.DNA_UNKNOWN: //Not expected
933    default:
934      builder.setCmdType(DatanodeCommandProto.Type.NullDatanodeCommand);
935    }
936    return builder.build();
937  }
938
939  public static KeyUpdateCommand convert(KeyUpdateCommandProto keyUpdateCmd) {
940    return new KeyUpdateCommand(PBHelper.convert(keyUpdateCmd.getKeys()));
941  }
942
943  public static FinalizeCommand convert(FinalizeCommandProto finalizeCmd) {
944    return new FinalizeCommand(finalizeCmd.getBlockPoolId());
945  }
946
947  public static BlockRecoveryCommand convert(
948      BlockRecoveryCommandProto recoveryCmd) {
949    List<RecoveringBlockProto> list = recoveryCmd.getBlocksList();
950    List<RecoveringBlock> recoveringBlocks = new ArrayList<RecoveringBlock>(
951        list.size());
952    
953    for (RecoveringBlockProto rbp : list) {
954      recoveringBlocks.add(PBHelper.convert(rbp));
955    }
956    return new BlockRecoveryCommand(recoveringBlocks);
957  }
958
959  public static BlockCommand convert(BlockCommandProto blkCmd) {
960    List<BlockProto> blockProtoList = blkCmd.getBlocksList();
961    Block[] blocks = new Block[blockProtoList.size()];
962    for (int i = 0; i < blockProtoList.size(); i++) {
963      blocks[i] = PBHelper.convert(blockProtoList.get(i));
964    }
965    List<DatanodeInfosProto> targetList = blkCmd.getTargetsList();
966    DatanodeInfo[][] targets = new DatanodeInfo[targetList.size()][];
967    for (int i = 0; i < targetList.size(); i++) {
968      targets[i] = PBHelper.convert(targetList.get(i));
969    }
970
971    List<StorageUuidsProto> targetStorageUuidsList = blkCmd.getTargetStorageUuidsList();
972    String[][] targetStorageIDs = new String[targetStorageUuidsList.size()][];
973    for(int i = 0; i < targetStorageIDs.length; i++) {
974      List<String> storageIDs = targetStorageUuidsList.get(i).getStorageUuidsList();
975      targetStorageIDs[i] = storageIDs.toArray(new String[storageIDs.size()]);
976    }
977
978    int action = DatanodeProtocol.DNA_UNKNOWN;
979    switch (blkCmd.getAction()) {
980    case TRANSFER:
981      action = DatanodeProtocol.DNA_TRANSFER;
982      break;
983    case INVALIDATE:
984      action = DatanodeProtocol.DNA_INVALIDATE;
985      break;
986    case SHUTDOWN:
987      action = DatanodeProtocol.DNA_SHUTDOWN;
988      break;
989    default:
990      throw new AssertionError("Unknown action type: " + blkCmd.getAction());
991    }
992    return new BlockCommand(action, blkCmd.getBlockPoolId(), blocks, targets,
993        targetStorageIDs);
994  }
995
996  public static BlockIdCommand convert(BlockIdCommandProto blkIdCmd) {
997    int numBlockIds = blkIdCmd.getBlockIdsCount();
998    long blockIds[] = new long[numBlockIds];
999    for (int i = 0; i < numBlockIds; i++) {
1000      blockIds[i] = blkIdCmd.getBlockIds(i);
1001    }
1002    int action = DatanodeProtocol.DNA_UNKNOWN;
1003    switch (blkIdCmd.getAction()) {
1004    case CACHE:
1005      action = DatanodeProtocol.DNA_CACHE;
1006      break;
1007    case UNCACHE:
1008      action = DatanodeProtocol.DNA_UNCACHE;
1009      break;
1010    default:
1011      throw new AssertionError("Unknown action type: " + blkIdCmd.getAction());
1012    }
1013    return new BlockIdCommand(action, blkIdCmd.getBlockPoolId(), blockIds);
1014  }
1015
1016  public static DatanodeInfo[] convert(DatanodeInfosProto datanodeInfosProto) {
1017    List<DatanodeInfoProto> proto = datanodeInfosProto.getDatanodesList();
1018    DatanodeInfo[] infos = new DatanodeInfo[proto.size()];
1019    for (int i = 0; i < infos.length; i++) {
1020      infos[i] = PBHelper.convert(proto.get(i));
1021    }
1022    return infos;
1023  }
1024
1025  public static BalancerBandwidthCommand convert(
1026      BalancerBandwidthCommandProto balancerCmd) {
1027    return new BalancerBandwidthCommand(balancerCmd.getBandwidth());
1028  }
1029
1030  public static ReceivedDeletedBlockInfoProto convert(
1031      ReceivedDeletedBlockInfo receivedDeletedBlockInfo) {
1032    ReceivedDeletedBlockInfoProto.Builder builder = 
1033        ReceivedDeletedBlockInfoProto.newBuilder();
1034    
1035    ReceivedDeletedBlockInfoProto.BlockStatus status;
1036    switch (receivedDeletedBlockInfo.getStatus()) {
1037    case RECEIVING_BLOCK:
1038      status = ReceivedDeletedBlockInfoProto.BlockStatus.RECEIVING;
1039      break;
1040    case RECEIVED_BLOCK:
1041      status = ReceivedDeletedBlockInfoProto.BlockStatus.RECEIVED;
1042      break;
1043    case DELETED_BLOCK:
1044      status = ReceivedDeletedBlockInfoProto.BlockStatus.DELETED;
1045      break;
1046    default:
1047      throw new IllegalArgumentException("Bad status: " +
1048          receivedDeletedBlockInfo.getStatus());
1049    }
1050    builder.setStatus(status);
1051    
1052    if (receivedDeletedBlockInfo.getDelHints() != null) {
1053      builder.setDeleteHint(receivedDeletedBlockInfo.getDelHints());
1054    }
1055    return builder.setBlock(PBHelper.convert(receivedDeletedBlockInfo.getBlock()))
1056        .build();
1057  }
1058
1059  public static ReceivedDeletedBlockInfo convert(
1060      ReceivedDeletedBlockInfoProto proto) {
1061    ReceivedDeletedBlockInfo.BlockStatus status = null;
1062    switch (proto.getStatus()) {
1063    case RECEIVING:
1064      status = BlockStatus.RECEIVING_BLOCK;
1065      break;
1066    case RECEIVED:
1067      status = BlockStatus.RECEIVED_BLOCK;
1068      break;
1069    case DELETED:
1070      status = BlockStatus.DELETED_BLOCK;
1071      break;
1072    }
1073    return new ReceivedDeletedBlockInfo(
1074        PBHelper.convert(proto.getBlock()),
1075        status,
1076        proto.hasDeleteHint() ? proto.getDeleteHint() : null);
1077  }
1078  
1079  public static NamespaceInfoProto convert(NamespaceInfo info) {
1080    return NamespaceInfoProto.newBuilder()
1081        .setBlockPoolID(info.getBlockPoolID())
1082        .setBuildVersion(info.getBuildVersion())
1083        .setUnused(0)
1084        .setStorageInfo(PBHelper.convert((StorageInfo)info))
1085        .setSoftwareVersion(info.getSoftwareVersion()).build();
1086  }
1087  
1088  // Located Block Arrays and Lists
1089  public static LocatedBlockProto[] convertLocatedBlock(LocatedBlock[] lb) {
1090    if (lb == null) return null;
1091    return convertLocatedBlock2(Arrays.asList(lb)).toArray(
1092        new LocatedBlockProto[lb.length]);
1093  }
1094  
1095  public static LocatedBlock[] convertLocatedBlock(LocatedBlockProto[] lb) {
1096    if (lb == null) return null;
1097    return convertLocatedBlock(Arrays.asList(lb)).toArray(
1098        new LocatedBlock[lb.length]);
1099  }
1100  
1101  public static List<LocatedBlock> convertLocatedBlock(
1102      List<LocatedBlockProto> lb) {
1103    if (lb == null) return null;
1104    final int len = lb.size();
1105    List<LocatedBlock> result = 
1106        new ArrayList<LocatedBlock>(len);
1107    for (int i = 0; i < len; ++i) {
1108      result.add(PBHelper.convert(lb.get(i)));
1109    }
1110    return result;
1111  }
1112  
1113  public static List<LocatedBlockProto> convertLocatedBlock2(List<LocatedBlock> lb) {
1114    if (lb == null) return null;
1115    final int len = lb.size();
1116    List<LocatedBlockProto> result = new ArrayList<LocatedBlockProto>(len);
1117    for (int i = 0; i < len; ++i) {
1118      result.add(PBHelper.convert(lb.get(i)));
1119    }
1120    return result;
1121  }
1122  
1123  
1124  // LocatedBlocks
1125  public static LocatedBlocks convert(LocatedBlocksProto lb) {
1126    return new LocatedBlocks(
1127        lb.getFileLength(), lb.getUnderConstruction(),
1128        PBHelper.convertLocatedBlock(lb.getBlocksList()),
1129        lb.hasLastBlock() ? PBHelper.convert(lb.getLastBlock()) : null,
1130        lb.getIsLastBlockComplete());
1131  }
1132  
1133  public static LocatedBlocksProto convert(LocatedBlocks lb) {
1134    if (lb == null) {
1135      return null;
1136    }
1137    LocatedBlocksProto.Builder builder = 
1138        LocatedBlocksProto.newBuilder();
1139    if (lb.getLastLocatedBlock() != null) {
1140      builder.setLastBlock(PBHelper.convert(lb.getLastLocatedBlock()));
1141    }
1142    return builder.setFileLength(lb.getFileLength())
1143        .setUnderConstruction(lb.isUnderConstruction())
1144        .addAllBlocks(PBHelper.convertLocatedBlock2(lb.getLocatedBlocks()))
1145        .setIsLastBlockComplete(lb.isLastBlockComplete()).build();
1146  }
1147  
1148  // DataEncryptionKey
1149  public static DataEncryptionKey convert(DataEncryptionKeyProto bet) {
1150    String encryptionAlgorithm = bet.getEncryptionAlgorithm();
1151    return new DataEncryptionKey(bet.getKeyId(),
1152        bet.getBlockPoolId(),
1153        bet.getNonce().toByteArray(),
1154        bet.getEncryptionKey().toByteArray(),
1155        bet.getExpiryDate(),
1156        encryptionAlgorithm.isEmpty() ? null : encryptionAlgorithm);
1157  }
1158  
1159  public static DataEncryptionKeyProto convert(DataEncryptionKey bet) {
1160    DataEncryptionKeyProto.Builder b = DataEncryptionKeyProto.newBuilder()
1161        .setKeyId(bet.keyId)
1162        .setBlockPoolId(bet.blockPoolId)
1163        .setNonce(ByteString.copyFrom(bet.nonce))
1164        .setEncryptionKey(ByteString.copyFrom(bet.encryptionKey))
1165        .setExpiryDate(bet.expiryDate);
1166    if (bet.encryptionAlgorithm != null) {
1167      b.setEncryptionAlgorithm(bet.encryptionAlgorithm);
1168    }
1169    return b.build();
1170  }
1171  
1172  public static FsServerDefaults convert(FsServerDefaultsProto fs) {
1173    if (fs == null) return null;
1174    return new FsServerDefaults(
1175        fs.getBlockSize(), fs.getBytesPerChecksum(), 
1176        fs.getWritePacketSize(), (short) fs.getReplication(),
1177        fs.getFileBufferSize(),
1178        fs.getEncryptDataTransfer(),
1179        fs.getTrashInterval(),
1180        PBHelper.convert(fs.getChecksumType()));
1181  }
1182  
1183  public static FsServerDefaultsProto convert(FsServerDefaults fs) {
1184    if (fs == null) return null;
1185    return FsServerDefaultsProto.newBuilder().
1186      setBlockSize(fs.getBlockSize()).
1187      setBytesPerChecksum(fs.getBytesPerChecksum()).
1188      setWritePacketSize(fs.getWritePacketSize())
1189      .setReplication(fs.getReplication())
1190      .setFileBufferSize(fs.getFileBufferSize())
1191      .setEncryptDataTransfer(fs.getEncryptDataTransfer())
1192      .setTrashInterval(fs.getTrashInterval())
1193      .setChecksumType(PBHelper.convert(fs.getChecksumType()))
1194      .build();
1195  }
1196  
1197  public static FsPermissionProto convert(FsPermission p) {
1198    return FsPermissionProto.newBuilder().setPerm(p.toExtendedShort()).build();
1199  }
1200  
1201  public static FsPermission convert(FsPermissionProto p) {
1202    return new FsAclPermission((short)p.getPerm());
1203  }
1204  
1205  
1206  // The creatFlag field in PB is a bitmask whose values are the same a the 
1207  // emum values of CreateFlag
1208  public static int convertCreateFlag(EnumSetWritable<CreateFlag> flag) {
1209    int value = 0;
1210    if (flag.contains(CreateFlag.APPEND)) {
1211      value |= CreateFlagProto.APPEND.getNumber();
1212    }
1213    if (flag.contains(CreateFlag.CREATE)) {
1214      value |= CreateFlagProto.CREATE.getNumber();
1215    }
1216    if (flag.contains(CreateFlag.OVERWRITE)) {
1217      value |= CreateFlagProto.OVERWRITE.getNumber();
1218    }
1219    return value;
1220  }
1221  
1222  public static EnumSetWritable<CreateFlag> convertCreateFlag(int flag) {
1223    EnumSet<CreateFlag> result = 
1224       EnumSet.noneOf(CreateFlag.class);   
1225    if ((flag & CreateFlagProto.APPEND_VALUE) == CreateFlagProto.APPEND_VALUE) {
1226      result.add(CreateFlag.APPEND);
1227    }
1228    if ((flag & CreateFlagProto.CREATE_VALUE) == CreateFlagProto.CREATE_VALUE) {
1229      result.add(CreateFlag.CREATE);
1230    }
1231    if ((flag & CreateFlagProto.OVERWRITE_VALUE) 
1232        == CreateFlagProto.OVERWRITE_VALUE) {
1233      result.add(CreateFlag.OVERWRITE);
1234    }
1235    return new EnumSetWritable<CreateFlag>(result);
1236  }
1237
1238  public static int convertCacheFlags(EnumSet<CacheFlag> flags) {
1239    int value = 0;
1240    if (flags.contains(CacheFlag.FORCE)) {
1241      value |= CacheFlagProto.FORCE.getNumber();
1242    }
1243    return value;
1244  }
1245
1246  public static EnumSet<CacheFlag> convertCacheFlags(int flags) {
1247    EnumSet<CacheFlag> result = EnumSet.noneOf(CacheFlag.class);
1248    if ((flags & CacheFlagProto.FORCE_VALUE) == CacheFlagProto.FORCE_VALUE) {
1249      result.add(CacheFlag.FORCE);
1250    }
1251    return result;
1252  }
1253
1254  public static HdfsFileStatus convert(HdfsFileStatusProto fs) {
1255    if (fs == null)
1256      return null;
1257    return new HdfsLocatedFileStatus(
1258        fs.getLength(), fs.getFileType().equals(FileType.IS_DIR), 
1259        fs.getBlockReplication(), fs.getBlocksize(),
1260        fs.getModificationTime(), fs.getAccessTime(),
1261        PBHelper.convert(fs.getPermission()), fs.getOwner(), fs.getGroup(), 
1262        fs.getFileType().equals(FileType.IS_SYMLINK) ? 
1263            fs.getSymlink().toByteArray() : null,
1264        fs.getPath().toByteArray(),
1265        fs.hasFileId()? fs.getFileId(): INodeId.GRANDFATHER_INODE_ID,
1266        fs.hasLocations() ? PBHelper.convert(fs.getLocations()) : null,
1267        fs.hasChildrenNum() ? fs.getChildrenNum() : -1);
1268  }
1269
1270  public static SnapshottableDirectoryStatus convert(
1271      SnapshottableDirectoryStatusProto sdirStatusProto) {
1272    if (sdirStatusProto == null) {
1273      return null;
1274    }
1275    final HdfsFileStatusProto status = sdirStatusProto.getDirStatus();
1276    return new SnapshottableDirectoryStatus(
1277        status.getModificationTime(),
1278        status.getAccessTime(),
1279        PBHelper.convert(status.getPermission()),
1280        status.getOwner(),
1281        status.getGroup(),
1282        status.getPath().toByteArray(),
1283        status.getFileId(),
1284        status.getChildrenNum(),
1285        sdirStatusProto.getSnapshotNumber(),
1286        sdirStatusProto.getSnapshotQuota(),
1287        sdirStatusProto.getParentFullpath().toByteArray());
1288  }
1289  
1290  public static HdfsFileStatusProto convert(HdfsFileStatus fs) {
1291    if (fs == null)
1292      return null;
1293    FileType fType = FileType.IS_FILE;
1294    if (fs.isDir()) {
1295      fType = FileType.IS_DIR;
1296    } else if (fs.isSymlink()) {
1297      fType = FileType.IS_SYMLINK;
1298    }
1299
1300    HdfsFileStatusProto.Builder builder = 
1301     HdfsFileStatusProto.newBuilder().
1302      setLength(fs.getLen()).
1303      setFileType(fType).
1304      setBlockReplication(fs.getReplication()).
1305      setBlocksize(fs.getBlockSize()).
1306      setModificationTime(fs.getModificationTime()).
1307      setAccessTime(fs.getAccessTime()).
1308      setPermission(PBHelper.convert(fs.getPermission())).
1309      setOwner(fs.getOwner()).
1310      setGroup(fs.getGroup()).
1311      setFileId(fs.getFileId()).
1312      setChildrenNum(fs.getChildrenNum()).
1313      setPath(ByteString.copyFrom(fs.getLocalNameInBytes()));
1314    if (fs.isSymlink())  {
1315      builder.setSymlink(ByteString.copyFrom(fs.getSymlinkInBytes()));
1316    }
1317    if (fs instanceof HdfsLocatedFileStatus) {
1318      LocatedBlocks locations = ((HdfsLocatedFileStatus)fs).getBlockLocations();
1319      if (locations != null) {
1320        builder.setLocations(PBHelper.convert(locations));
1321      }
1322    }
1323    return builder.build();
1324  }
1325  
1326  public static SnapshottableDirectoryStatusProto convert(
1327      SnapshottableDirectoryStatus status) {
1328    if (status == null) {
1329      return null;
1330    }
1331    int snapshotNumber = status.getSnapshotNumber();
1332    int snapshotQuota = status.getSnapshotQuota();
1333    byte[] parentFullPath = status.getParentFullPath();
1334    ByteString parentFullPathBytes = ByteString.copyFrom(
1335        parentFullPath == null ? DFSUtil.EMPTY_BYTES : parentFullPath);
1336    HdfsFileStatusProto fs = convert(status.getDirStatus());
1337    SnapshottableDirectoryStatusProto.Builder builder = 
1338        SnapshottableDirectoryStatusProto
1339        .newBuilder().setSnapshotNumber(snapshotNumber)
1340        .setSnapshotQuota(snapshotQuota).setParentFullpath(parentFullPathBytes)
1341        .setDirStatus(fs);
1342    return builder.build();
1343  }
1344  
1345  public static HdfsFileStatusProto[] convert(HdfsFileStatus[] fs) {
1346    if (fs == null) return null;
1347    final int len = fs.length;
1348    HdfsFileStatusProto[] result = new HdfsFileStatusProto[len];
1349    for (int i = 0; i < len; ++i) {
1350      result[i] = PBHelper.convert(fs[i]);
1351    }
1352    return result;
1353  }
1354  
1355  public static HdfsFileStatus[] convert(HdfsFileStatusProto[] fs) {
1356    if (fs == null) return null;
1357    final int len = fs.length;
1358    HdfsFileStatus[] result = new HdfsFileStatus[len];
1359    for (int i = 0; i < len; ++i) {
1360      result[i] = PBHelper.convert(fs[i]);
1361    }
1362    return result;
1363  }
1364  
1365  public static DirectoryListing convert(DirectoryListingProto dl) {
1366    if (dl == null)
1367      return null;
1368    List<HdfsFileStatusProto> partList =  dl.getPartialListingList();
1369    return new DirectoryListing( 
1370        partList.isEmpty() ? new HdfsLocatedFileStatus[0] 
1371          : PBHelper.convert(
1372              partList.toArray(new HdfsFileStatusProto[partList.size()])),
1373        dl.getRemainingEntries());
1374  }
1375
1376  public static DirectoryListingProto convert(DirectoryListing d) {
1377    if (d == null)
1378      return null;
1379    return DirectoryListingProto.newBuilder().
1380        addAllPartialListing(Arrays.asList(
1381            PBHelper.convert(d.getPartialListing()))).
1382        setRemainingEntries(d.getRemainingEntries()).
1383        build();
1384  }
1385
1386  public static long[] convert(GetFsStatsResponseProto res) {
1387    long[] result = new long[6];
1388    result[ClientProtocol.GET_STATS_CAPACITY_IDX] = res.getCapacity();
1389    result[ClientProtocol.GET_STATS_USED_IDX] = res.getUsed();
1390    result[ClientProtocol.GET_STATS_REMAINING_IDX] = res.getRemaining();
1391    result[ClientProtocol.GET_STATS_UNDER_REPLICATED_IDX] = res.getUnderReplicated();
1392    result[ClientProtocol.GET_STATS_CORRUPT_BLOCKS_IDX] = res.getCorruptBlocks();
1393    result[ClientProtocol.GET_STATS_MISSING_BLOCKS_IDX] = res.getMissingBlocks();
1394    return result;
1395  }
1396  
1397  public static GetFsStatsResponseProto convert(long[] fsStats) {
1398    GetFsStatsResponseProto.Builder result = GetFsStatsResponseProto
1399        .newBuilder();
1400    if (fsStats.length >= ClientProtocol.GET_STATS_CAPACITY_IDX + 1)
1401      result.setCapacity(fsStats[ClientProtocol.GET_STATS_CAPACITY_IDX]);
1402    if (fsStats.length >= ClientProtocol.GET_STATS_USED_IDX + 1)
1403      result.setUsed(fsStats[ClientProtocol.GET_STATS_USED_IDX]);
1404    if (fsStats.length >= ClientProtocol.GET_STATS_REMAINING_IDX + 1)
1405      result.setRemaining(fsStats[ClientProtocol.GET_STATS_REMAINING_IDX]);
1406    if (fsStats.length >= ClientProtocol.GET_STATS_UNDER_REPLICATED_IDX + 1)
1407      result.setUnderReplicated(
1408              fsStats[ClientProtocol.GET_STATS_UNDER_REPLICATED_IDX]);
1409    if (fsStats.length >= ClientProtocol.GET_STATS_CORRUPT_BLOCKS_IDX + 1)
1410      result.setCorruptBlocks(
1411          fsStats[ClientProtocol.GET_STATS_CORRUPT_BLOCKS_IDX]);
1412    if (fsStats.length >= ClientProtocol.GET_STATS_MISSING_BLOCKS_IDX + 1)
1413      result.setMissingBlocks(
1414          fsStats[ClientProtocol.GET_STATS_MISSING_BLOCKS_IDX]);
1415    return result.build();
1416  }
1417  
1418  public static DatanodeReportTypeProto
1419    convert(DatanodeReportType t) {
1420    switch (t) {
1421    case ALL: return DatanodeReportTypeProto.ALL;
1422    case LIVE: return DatanodeReportTypeProto.LIVE;
1423    case DEAD: return DatanodeReportTypeProto.DEAD;
1424    case DECOMMISSIONING: return DatanodeReportTypeProto.DECOMMISSIONING;
1425    default: 
1426      throw new IllegalArgumentException("Unexpected data type report:" + t);
1427    }
1428  }
1429  
1430  public static DatanodeReportType 
1431    convert(DatanodeReportTypeProto t) {
1432    switch (t) {
1433    case ALL: return DatanodeReportType.ALL;
1434    case LIVE: return DatanodeReportType.LIVE;
1435    case DEAD: return DatanodeReportType.DEAD;
1436    case DECOMMISSIONING: return DatanodeReportType.DECOMMISSIONING;
1437    default: 
1438      throw new IllegalArgumentException("Unexpected data type report:" + t);
1439    }
1440  }
1441
1442  public static SafeModeActionProto convert(
1443      SafeModeAction a) {
1444    switch (a) {
1445    case SAFEMODE_LEAVE:
1446      return SafeModeActionProto.SAFEMODE_LEAVE;
1447    case SAFEMODE_ENTER:
1448      return SafeModeActionProto.SAFEMODE_ENTER;
1449    case SAFEMODE_GET:
1450      return SafeModeActionProto.SAFEMODE_GET;
1451    default:
1452      throw new IllegalArgumentException("Unexpected SafeModeAction :" + a);
1453    }
1454  }
1455  
1456  public static SafeModeAction convert(
1457      ClientNamenodeProtocolProtos.SafeModeActionProto a) {
1458    switch (a) {
1459    case SAFEMODE_LEAVE:
1460      return SafeModeAction.SAFEMODE_LEAVE;
1461    case SAFEMODE_ENTER:
1462      return SafeModeAction.SAFEMODE_ENTER;
1463    case SAFEMODE_GET:
1464      return SafeModeAction.SAFEMODE_GET;
1465    default:
1466      throw new IllegalArgumentException("Unexpected SafeModeAction :" + a);
1467    }
1468  }
1469  
1470  public static RollingUpgradeActionProto convert(RollingUpgradeAction a) {
1471    switch (a) {
1472    case QUERY:
1473      return RollingUpgradeActionProto.QUERY;
1474    case PREPARE:
1475      return RollingUpgradeActionProto.START;
1476    case FINALIZE:
1477      return RollingUpgradeActionProto.FINALIZE;
1478    default:
1479      throw new IllegalArgumentException("Unexpected value: " + a);
1480    }
1481  }
1482  
1483  public static RollingUpgradeAction convert(RollingUpgradeActionProto a) {
1484    switch (a) {
1485    case QUERY:
1486      return RollingUpgradeAction.QUERY;
1487    case START:
1488      return RollingUpgradeAction.PREPARE;
1489    case FINALIZE:
1490      return RollingUpgradeAction.FINALIZE;
1491    default:
1492      throw new IllegalArgumentException("Unexpected value: " + a);
1493    }
1494  }
1495
1496  public static RollingUpgradeStatusProto convertRollingUpgradeStatus(
1497      RollingUpgradeStatus status) {
1498    return RollingUpgradeStatusProto.newBuilder()
1499        .setBlockPoolId(status.getBlockPoolId())
1500        .build();
1501  }
1502
1503  public static RollingUpgradeStatus convert(RollingUpgradeStatusProto proto) {
1504    return new RollingUpgradeStatus(proto.getBlockPoolId());
1505  }
1506
1507  public static RollingUpgradeInfoProto convert(RollingUpgradeInfo info) {
1508    return RollingUpgradeInfoProto.newBuilder()
1509        .setStatus(convertRollingUpgradeStatus(info))
1510        .setCreatedRollbackImages(info.createdRollbackImages())
1511        .setStartTime(info.getStartTime())
1512        .setFinalizeTime(info.getFinalizeTime())
1513        .build();
1514  }
1515
1516  public static RollingUpgradeInfo convert(RollingUpgradeInfoProto proto) {
1517    RollingUpgradeStatusProto status = proto.getStatus();
1518    return new RollingUpgradeInfo(status.getBlockPoolId(),
1519        proto.getCreatedRollbackImages(),
1520        proto.getStartTime(), proto.getFinalizeTime());
1521  }
1522
1523  public static CorruptFileBlocks convert(CorruptFileBlocksProto c) {
1524    if (c == null)
1525      return null;
1526    List<String> fileList = c.getFilesList();
1527    return new CorruptFileBlocks(fileList.toArray(new String[fileList.size()]),
1528        c.getCookie());
1529  }
1530
1531  public static CorruptFileBlocksProto convert(CorruptFileBlocks c) {
1532    if (c == null)
1533      return null;
1534    return CorruptFileBlocksProto.newBuilder().
1535        addAllFiles(Arrays.asList(c.getFiles())).
1536        setCookie(c.getCookie()).
1537        build();
1538  }
1539  
1540  public static ContentSummary convert(ContentSummaryProto cs) {
1541    if (cs == null) return null;
1542    return new ContentSummary(
1543      cs.getLength(), cs.getFileCount(), cs.getDirectoryCount(), cs.getQuota(),
1544      cs.getSpaceConsumed(), cs.getSpaceQuota());
1545  }
1546  
1547  public static ContentSummaryProto convert(ContentSummary cs) {
1548    if (cs == null) return null;
1549    return ContentSummaryProto.newBuilder().
1550        setLength(cs.getLength()).
1551        setFileCount(cs.getFileCount()).
1552        setDirectoryCount(cs.getDirectoryCount()).
1553        setQuota(cs.getQuota()).
1554        setSpaceConsumed(cs.getSpaceConsumed()).
1555        setSpaceQuota(cs.getSpaceQuota()).
1556        build();
1557  }
1558
1559  public static NNHAStatusHeartbeat convert(NNHAStatusHeartbeatProto s) {
1560    if (s == null) return null;
1561    switch (s.getState()) {
1562    case ACTIVE:
1563      return new NNHAStatusHeartbeat(HAServiceState.ACTIVE, s.getTxid());
1564    case STANDBY:
1565      return new NNHAStatusHeartbeat(HAServiceState.STANDBY, s.getTxid());
1566    default:
1567      throw new IllegalArgumentException("Unexpected NNHAStatusHeartbeat.State:" + s.getState());
1568    }
1569  }
1570
1571  public static NNHAStatusHeartbeatProto convert(NNHAStatusHeartbeat hb) {
1572    if (hb == null) return null;
1573    NNHAStatusHeartbeatProto.Builder builder =
1574      NNHAStatusHeartbeatProto.newBuilder();
1575    switch (hb.getState()) {
1576      case ACTIVE:
1577        builder.setState(NNHAStatusHeartbeatProto.State.ACTIVE);
1578        break;
1579      case STANDBY:
1580        builder.setState(NNHAStatusHeartbeatProto.State.STANDBY);
1581        break;
1582      default:
1583        throw new IllegalArgumentException("Unexpected NNHAStatusHeartbeat.State:" +
1584            hb.getState());
1585    }
1586    builder.setTxid(hb.getTxId());
1587    return builder.build();
1588  }
1589
1590  public static DatanodeStorageProto convert(DatanodeStorage s) {
1591    return DatanodeStorageProto.newBuilder()
1592        .setState(PBHelper.convertState(s.getState()))
1593        .setStorageType(PBHelper.convertStorageType(s.getStorageType()))
1594        .setStorageUuid(s.getStorageID()).build();
1595  }
1596
1597  private static StorageState convertState(State state) {
1598    switch(state) {
1599    case READ_ONLY_SHARED:
1600      return StorageState.READ_ONLY_SHARED;
1601    case NORMAL:
1602    default:
1603      return StorageState.NORMAL;
1604    }
1605  }
1606
1607  private static StorageTypeProto convertStorageType(
1608      StorageType type) {
1609    switch(type) {
1610    case DISK:
1611      return StorageTypeProto.DISK;
1612    case SSD:
1613      return StorageTypeProto.SSD;
1614    default:
1615      throw new IllegalStateException(
1616          "BUG: StorageType not found, type=" + type);
1617    }
1618  }
1619
1620  public static DatanodeStorage convert(DatanodeStorageProto s) {
1621    return new DatanodeStorage(s.getStorageUuid(),
1622                               PBHelper.convertState(s.getState()),
1623                               PBHelper.convertType(s.getStorageType()));
1624  }
1625
1626  private static State convertState(StorageState state) {
1627    switch(state) {
1628    case READ_ONLY_SHARED:
1629      return DatanodeStorage.State.READ_ONLY_SHARED;
1630    case NORMAL:
1631    default:
1632      return DatanodeStorage.State.NORMAL;
1633    }
1634  }
1635
1636  private static StorageType convertType(StorageTypeProto type) {
1637    switch(type) {
1638      case DISK:
1639        return StorageType.DISK;
1640      case SSD:
1641        return StorageType.SSD;
1642      default:
1643        throw new IllegalStateException(
1644            "BUG: StorageTypeProto not found, type=" + type);
1645    }
1646  }
1647
1648  private static StorageType[] convertStorageTypeProtos(
1649      List<StorageTypeProto> storageTypesList) {
1650    final StorageType[] storageTypes = new StorageType[storageTypesList.size()];
1651    for (int i = 0; i < storageTypes.length; ++i) {
1652      storageTypes[i] = PBHelper.convertType(storageTypesList.get(i));
1653    }
1654    return storageTypes;
1655  }
1656
1657  public static StorageReportProto convert(StorageReport r) {
1658    StorageReportProto.Builder builder = StorageReportProto.newBuilder()
1659        .setBlockPoolUsed(r.getBlockPoolUsed()).setCapacity(r.getCapacity())
1660        .setDfsUsed(r.getDfsUsed()).setRemaining(r.getRemaining())
1661        .setStorageUuid(r.getStorage().getStorageID())
1662        .setStorage(convert(r.getStorage()));
1663    return builder.build();
1664  }
1665
1666  public static StorageReport convert(StorageReportProto p) {
1667    return new StorageReport(
1668        p.hasStorage() ?
1669            convert(p.getStorage()) :
1670            new DatanodeStorage(p.getStorageUuid()),
1671        p.getFailed(), p.getCapacity(), p.getDfsUsed(), p.getRemaining(),
1672        p.getBlockPoolUsed());
1673  }
1674
1675  public static StorageReport[] convertStorageReports(
1676      List<StorageReportProto> list) {
1677    final StorageReport[] report = new StorageReport[list.size()];
1678    for (int i = 0; i < report.length; i++) {
1679      report[i] = convert(list.get(i));
1680    }
1681    return report;
1682  }
1683
1684  public static JournalInfo convert(JournalInfoProto info) {
1685    int lv = info.hasLayoutVersion() ? info.getLayoutVersion() : 0;
1686    int nsID = info.hasNamespaceID() ? info.getNamespaceID() : 0;
1687    return new JournalInfo(lv, info.getClusterID(), nsID);
1688  }
1689
1690  /**
1691   * Method used for converting {@link JournalInfoProto} sent from Namenode
1692   * to Journal receivers to {@link NamenodeRegistration}.
1693   */
1694  public static JournalInfoProto convert(JournalInfo j) {
1695    return JournalInfoProto.newBuilder().setClusterID(j.getClusterId())
1696        .setLayoutVersion(j.getLayoutVersion())
1697        .setNamespaceID(j.getNamespaceId()).build();
1698  } 
1699  
1700  public static SnapshottableDirectoryStatus[] convert(
1701      SnapshottableDirectoryListingProto sdlp) {
1702    if (sdlp == null)
1703      return null;
1704    List<SnapshottableDirectoryStatusProto> list = sdlp
1705        .getSnapshottableDirListingList();
1706    if (list.isEmpty()) {
1707      return new SnapshottableDirectoryStatus[0];
1708    } else {
1709      SnapshottableDirectoryStatus[] result = 
1710          new SnapshottableDirectoryStatus[list.size()];
1711      for (int i = 0; i < list.size(); i++) {
1712        result[i] = PBHelper.convert(list.get(i));
1713      }
1714      return result;
1715    }
1716  }
1717  
1718  public static SnapshottableDirectoryListingProto convert(
1719      SnapshottableDirectoryStatus[] status) {
1720    if (status == null)
1721      return null;
1722    SnapshottableDirectoryStatusProto[] protos = 
1723        new SnapshottableDirectoryStatusProto[status.length];
1724    for (int i = 0; i < status.length; i++) {
1725      protos[i] = PBHelper.convert(status[i]);
1726    }
1727    List<SnapshottableDirectoryStatusProto> protoList = Arrays.asList(protos);
1728    return SnapshottableDirectoryListingProto.newBuilder()
1729        .addAllSnapshottableDirListing(protoList).build();
1730  }
1731  
1732  public static DiffReportEntry convert(SnapshotDiffReportEntryProto entry) {
1733    if (entry == null) {
1734      return null;
1735    }
1736    DiffType type = DiffType.getTypeFromLabel(entry
1737        .getModificationLabel());
1738    return type == null ? null : new DiffReportEntry(type, entry.getFullpath()
1739        .toByteArray(), entry.hasTargetPath() ? entry.getTargetPath()
1740        .toByteArray() : null);
1741  }
1742  
1743  public static SnapshotDiffReportEntryProto convert(DiffReportEntry entry) {
1744    if (entry == null) {
1745      return null;
1746    }
1747    ByteString sourcePath = ByteString
1748        .copyFrom(entry.getSourcePath() == null ? DFSUtil.EMPTY_BYTES : entry
1749            .getSourcePath());
1750    String modification = entry.getType().getLabel();
1751    SnapshotDiffReportEntryProto.Builder builder = SnapshotDiffReportEntryProto
1752        .newBuilder().setFullpath(sourcePath)
1753        .setModificationLabel(modification);
1754    if (entry.getType() == DiffType.RENAME) {
1755      ByteString targetPath = ByteString
1756          .copyFrom(entry.getTargetPath() == null ? DFSUtil.EMPTY_BYTES : entry
1757              .getTargetPath());
1758      builder.setTargetPath(targetPath);
1759    }
1760    return builder.build();
1761  }
1762  
1763  public static SnapshotDiffReport convert(SnapshotDiffReportProto reportProto) {
1764    if (reportProto == null) {
1765      return null;
1766    }
1767    String snapshotDir = reportProto.getSnapshotRoot();
1768    String fromSnapshot = reportProto.getFromSnapshot();
1769    String toSnapshot = reportProto.getToSnapshot();
1770    List<SnapshotDiffReportEntryProto> list = reportProto
1771        .getDiffReportEntriesList();
1772    List<DiffReportEntry> entries = new ArrayList<DiffReportEntry>();
1773    for (SnapshotDiffReportEntryProto entryProto : list) {
1774      DiffReportEntry entry = convert(entryProto);
1775      if (entry != null)
1776        entries.add(entry);
1777    }
1778    return new SnapshotDiffReport(snapshotDir, fromSnapshot, toSnapshot,
1779        entries);
1780  }
1781  
1782  public static SnapshotDiffReportProto convert(SnapshotDiffReport report) {
1783    if (report == null) {
1784      return null;
1785    }
1786    List<DiffReportEntry> entries = report.getDiffList();
1787    List<SnapshotDiffReportEntryProto> entryProtos = 
1788        new ArrayList<SnapshotDiffReportEntryProto>();
1789    for (DiffReportEntry entry : entries) {
1790      SnapshotDiffReportEntryProto entryProto = convert(entry);
1791      if (entryProto != null)
1792        entryProtos.add(entryProto);
1793    }
1794    
1795    SnapshotDiffReportProto reportProto = SnapshotDiffReportProto.newBuilder()
1796        .setSnapshotRoot(report.getSnapshotRoot())
1797        .setFromSnapshot(report.getFromSnapshot())
1798        .setToSnapshot(report.getLaterSnapshotName())
1799        .addAllDiffReportEntries(entryProtos).build();
1800    return reportProto;
1801  }
1802
1803  public static DataChecksum.Type convert(HdfsProtos.ChecksumTypeProto type) {
1804    return DataChecksum.Type.valueOf(type.getNumber());
1805  }
1806
1807  public static CacheDirectiveInfoProto convert
1808      (CacheDirectiveInfo info) {
1809    CacheDirectiveInfoProto.Builder builder = 
1810        CacheDirectiveInfoProto.newBuilder();
1811    if (info.getId() != null) {
1812      builder.setId(info.getId());
1813    }
1814    if (info.getPath() != null) {
1815      builder.setPath(info.getPath().toUri().getPath());
1816    }
1817    if (info.getReplication() != null) {
1818      builder.setReplication(info.getReplication());
1819    }
1820    if (info.getPool() != null) {
1821      builder.setPool(info.getPool());
1822    }
1823    if (info.getExpiration() != null) {
1824      builder.setExpiration(convert(info.getExpiration()));
1825    }
1826    return builder.build();
1827  }
1828
1829  public static CacheDirectiveInfo convert
1830      (CacheDirectiveInfoProto proto) {
1831    CacheDirectiveInfo.Builder builder =
1832        new CacheDirectiveInfo.Builder();
1833    if (proto.hasId()) {
1834      builder.setId(proto.getId());
1835    }
1836    if (proto.hasPath()) {
1837      builder.setPath(new Path(proto.getPath()));
1838    }
1839    if (proto.hasReplication()) {
1840      builder.setReplication(Shorts.checkedCast(
1841          proto.getReplication()));
1842    }
1843    if (proto.hasPool()) {
1844      builder.setPool(proto.getPool());
1845    }
1846    if (proto.hasExpiration()) {
1847      builder.setExpiration(convert(proto.getExpiration()));
1848    }
1849    return builder.build();
1850  }
1851
1852  public static CacheDirectiveInfoExpirationProto convert(
1853      CacheDirectiveInfo.Expiration expiration) {
1854    return CacheDirectiveInfoExpirationProto.newBuilder()
1855        .setIsRelative(expiration.isRelative())
1856        .setMillis(expiration.getMillis())
1857        .build();
1858  }
1859
1860  public static CacheDirectiveInfo.Expiration convert(
1861      CacheDirectiveInfoExpirationProto proto) {
1862    if (proto.getIsRelative()) {
1863      return CacheDirectiveInfo.Expiration.newRelative(proto.getMillis());
1864    }
1865    return CacheDirectiveInfo.Expiration.newAbsolute(proto.getMillis());
1866  }
1867
1868  public static CacheDirectiveStatsProto convert(CacheDirectiveStats stats) {
1869    CacheDirectiveStatsProto.Builder builder = 
1870        CacheDirectiveStatsProto.newBuilder();
1871    builder.setBytesNeeded(stats.getBytesNeeded());
1872    builder.setBytesCached(stats.getBytesCached());
1873    builder.setFilesNeeded(stats.getFilesNeeded());
1874    builder.setFilesCached(stats.getFilesCached());
1875    builder.setHasExpired(stats.hasExpired());
1876    return builder.build();
1877  }
1878  
1879  public static CacheDirectiveStats convert(CacheDirectiveStatsProto proto) {
1880    CacheDirectiveStats.Builder builder = new CacheDirectiveStats.Builder();
1881    builder.setBytesNeeded(proto.getBytesNeeded());
1882    builder.setBytesCached(proto.getBytesCached());
1883    builder.setFilesNeeded(proto.getFilesNeeded());
1884    builder.setFilesCached(proto.getFilesCached());
1885    builder.setHasExpired(proto.getHasExpired());
1886    return builder.build();
1887  }
1888
1889  public static CacheDirectiveEntryProto convert(CacheDirectiveEntry entry) {
1890    CacheDirectiveEntryProto.Builder builder = 
1891        CacheDirectiveEntryProto.newBuilder();
1892    builder.setInfo(PBHelper.convert(entry.getInfo()));
1893    builder.setStats(PBHelper.convert(entry.getStats()));
1894    return builder.build();
1895  }
1896  
1897  public static CacheDirectiveEntry convert(CacheDirectiveEntryProto proto) {
1898    CacheDirectiveInfo info = PBHelper.convert(proto.getInfo());
1899    CacheDirectiveStats stats = PBHelper.convert(proto.getStats());
1900    return new CacheDirectiveEntry(info, stats);
1901  }
1902
1903  public static CachePoolInfoProto convert(CachePoolInfo info) {
1904    CachePoolInfoProto.Builder builder = CachePoolInfoProto.newBuilder();
1905    builder.setPoolName(info.getPoolName());
1906    if (info.getOwnerName() != null) {
1907      builder.setOwnerName(info.getOwnerName());
1908    }
1909    if (info.getGroupName() != null) {
1910      builder.setGroupName(info.getGroupName());
1911    }
1912    if (info.getMode() != null) {
1913      builder.setMode(info.getMode().toShort());
1914    }
1915    if (info.getLimit() != null) {
1916      builder.setLimit(info.getLimit());
1917    }
1918    if (info.getMaxRelativeExpiryMs() != null) {
1919      builder.setMaxRelativeExpiry(info.getMaxRelativeExpiryMs());
1920    }
1921    return builder.build();
1922  }
1923
1924  public static CachePoolInfo convert (CachePoolInfoProto proto) {
1925    // Pool name is a required field, the rest are optional
1926    String poolName = checkNotNull(proto.getPoolName());
1927    CachePoolInfo info = new CachePoolInfo(poolName);
1928    if (proto.hasOwnerName()) {
1929        info.setOwnerName(proto.getOwnerName());
1930    }
1931    if (proto.hasGroupName()) {
1932      info.setGroupName(proto.getGroupName());
1933    }
1934    if (proto.hasMode()) {
1935      info.setMode(new FsPermission((short)proto.getMode()));
1936    }
1937    if (proto.hasLimit())  {
1938      info.setLimit(proto.getLimit());
1939    }
1940    if (proto.hasMaxRelativeExpiry()) {
1941      info.setMaxRelativeExpiryMs(proto.getMaxRelativeExpiry());
1942    }
1943    return info;
1944  }
1945
1946  public static CachePoolStatsProto convert(CachePoolStats stats) {
1947    CachePoolStatsProto.Builder builder = CachePoolStatsProto.newBuilder();
1948    builder.setBytesNeeded(stats.getBytesNeeded());
1949    builder.setBytesCached(stats.getBytesCached());
1950    builder.setBytesOverlimit(stats.getBytesOverlimit());
1951    builder.setFilesNeeded(stats.getFilesNeeded());
1952    builder.setFilesCached(stats.getFilesCached());
1953    return builder.build();
1954  }
1955
1956  public static CachePoolStats convert (CachePoolStatsProto proto) {
1957    CachePoolStats.Builder builder = new CachePoolStats.Builder();
1958    builder.setBytesNeeded(proto.getBytesNeeded());
1959    builder.setBytesCached(proto.getBytesCached());
1960    builder.setBytesOverlimit(proto.getBytesOverlimit());
1961    builder.setFilesNeeded(proto.getFilesNeeded());
1962    builder.setFilesCached(proto.getFilesCached());
1963    return builder.build();
1964  }
1965
1966  public static CachePoolEntryProto convert(CachePoolEntry entry) {
1967    CachePoolEntryProto.Builder builder = CachePoolEntryProto.newBuilder();
1968    builder.setInfo(PBHelper.convert(entry.getInfo()));
1969    builder.setStats(PBHelper.convert(entry.getStats()));
1970    return builder.build();
1971  }
1972
1973  public static CachePoolEntry convert (CachePoolEntryProto proto) {
1974    CachePoolInfo info = PBHelper.convert(proto.getInfo());
1975    CachePoolStats stats = PBHelper.convert(proto.getStats());
1976    return new CachePoolEntry(info, stats);
1977  }
1978  
1979  public static HdfsProtos.ChecksumTypeProto convert(DataChecksum.Type type) {
1980    return HdfsProtos.ChecksumTypeProto.valueOf(type.id);
1981  }
1982
1983  public static DatanodeLocalInfoProto convert(DatanodeLocalInfo info) {
1984    DatanodeLocalInfoProto.Builder builder = DatanodeLocalInfoProto.newBuilder();
1985    builder.setSoftwareVersion(info.getSoftwareVersion());
1986    builder.setConfigVersion(info.getConfigVersion());
1987    builder.setUptime(info.getUptime());
1988    return builder.build();
1989  }
1990
1991  public static DatanodeLocalInfo convert(DatanodeLocalInfoProto proto) {
1992    return new DatanodeLocalInfo(proto.getSoftwareVersion(),
1993        proto.getConfigVersion(), proto.getUptime());
1994  }
1995
1996  public static InputStream vintPrefixed(final InputStream input)
1997      throws IOException {
1998    final int firstByte = input.read();
1999    if (firstByte == -1) {
2000      throw new EOFException("Premature EOF: no length prefix available");
2001    }
2002
2003    int size = CodedInputStream.readRawVarint32(firstByte, input);
2004    assert size >= 0;
2005    return new ExactSizeInputStream(input, size);
2006  }
2007
2008  private static AclEntryScopeProto convert(AclEntryScope v) {
2009    return AclEntryScopeProto.valueOf(v.ordinal());
2010  }
2011
2012  private static AclEntryScope convert(AclEntryScopeProto v) {
2013    return castEnum(v, ACL_ENTRY_SCOPE_VALUES);
2014  }
2015
2016  private static AclEntryTypeProto convert(AclEntryType e) {
2017    return AclEntryTypeProto.valueOf(e.ordinal());
2018  }
2019
2020  private static AclEntryType convert(AclEntryTypeProto v) {
2021    return castEnum(v, ACL_ENTRY_TYPE_VALUES);
2022  }
2023  
2024  private static XAttrNamespaceProto convert(XAttr.NameSpace v) {
2025    return XAttrNamespaceProto.valueOf(v.ordinal());
2026  }
2027  
2028  private static XAttr.NameSpace convert(XAttrNamespaceProto v) {
2029    return castEnum(v, XATTR_NAMESPACE_VALUES);
2030  }
2031
2032  private static FsActionProto convert(FsAction v) {
2033    return FsActionProto.valueOf(v != null ? v.ordinal() : 0);
2034  }
2035
2036  private static FsAction convert(FsActionProto v) {
2037    return castEnum(v, FSACTION_VALUES);
2038  }
2039
2040  public static List<AclEntryProto> convertAclEntryProto(
2041      List<AclEntry> aclSpec) {
2042    ArrayList<AclEntryProto> r = Lists.newArrayListWithCapacity(aclSpec.size());
2043    for (AclEntry e : aclSpec) {
2044      AclEntryProto.Builder builder = AclEntryProto.newBuilder();
2045      builder.setType(convert(e.getType()));
2046      builder.setScope(convert(e.getScope()));
2047      builder.setPermissions(convert(e.getPermission()));
2048      if (e.getName() != null) {
2049        builder.setName(e.getName());
2050      }
2051      r.add(builder.build());
2052    }
2053    return r;
2054  }
2055
2056  public static List<AclEntry> convertAclEntry(List<AclEntryProto> aclSpec) {
2057    ArrayList<AclEntry> r = Lists.newArrayListWithCapacity(aclSpec.size());
2058    for (AclEntryProto e : aclSpec) {
2059      AclEntry.Builder builder = new AclEntry.Builder();
2060      builder.setType(convert(e.getType()));
2061      builder.setScope(convert(e.getScope()));
2062      builder.setPermission(convert(e.getPermissions()));
2063      if (e.hasName()) {
2064        builder.setName(e.getName());
2065      }
2066      r.add(builder.build());
2067    }
2068    return r;
2069  }
2070
2071  public static AclStatus convert(GetAclStatusResponseProto e) {
2072    AclStatusProto r = e.getResult();
2073    return new AclStatus.Builder().owner(r.getOwner()).group(r.getGroup())
2074        .stickyBit(r.getSticky())
2075        .addEntries(convertAclEntry(r.getEntriesList())).build();
2076  }
2077
2078  public static GetAclStatusResponseProto convert(AclStatus e) {
2079    AclStatusProto r = AclStatusProto.newBuilder().setOwner(e.getOwner())
2080        .setGroup(e.getGroup()).setSticky(e.isStickyBit())
2081        .addAllEntries(convertAclEntryProto(e.getEntries())).build();
2082    return GetAclStatusResponseProto.newBuilder().setResult(r).build();
2083  }
2084  
2085  public static XAttrProto convertXAttrProto(XAttr a) {
2086    XAttrProto.Builder builder = XAttrProto.newBuilder();
2087    builder.setNamespace(convert(a.getNameSpace()));
2088    if (a.getName() != null) {
2089      builder.setName(a.getName());
2090    }
2091    if (a.getValue() != null) {
2092      builder.setValue(getByteString(a.getValue()));
2093    }
2094    return builder.build();
2095  }
2096  
2097  public static List<XAttrProto> convertXAttrProto(
2098      List<XAttr> xAttrSpec) {
2099    if (xAttrSpec == null) {
2100      return Lists.newArrayListWithCapacity(0);
2101    }
2102    ArrayList<XAttrProto> xAttrs = Lists.newArrayListWithCapacity(
2103        xAttrSpec.size());
2104    for (XAttr a : xAttrSpec) {
2105      XAttrProto.Builder builder = XAttrProto.newBuilder();
2106      builder.setNamespace(convert(a.getNameSpace()));
2107      if (a.getName() != null) {
2108        builder.setName(a.getName());
2109      }
2110      if (a.getValue() != null) {
2111        builder.setValue(getByteString(a.getValue()));
2112      }
2113      xAttrs.add(builder.build());
2114    }
2115    return xAttrs;
2116  }
2117  
2118  /**
2119   * The flag field in PB is a bitmask whose values are the same a the 
2120   * emum values of XAttrSetFlag
2121   */
2122  public static int convert(EnumSet<XAttrSetFlag> flag) {
2123    int value = 0;
2124    if (flag.contains(XAttrSetFlag.CREATE)) {
2125      value |= XAttrSetFlagProto.XATTR_CREATE.getNumber();
2126    }
2127    if (flag.contains(XAttrSetFlag.REPLACE)) {
2128      value |= XAttrSetFlagProto.XATTR_REPLACE.getNumber();
2129    }
2130    return value;
2131  }
2132 
2133  public static EnumSet<XAttrSetFlag> convert(int flag) {
2134    EnumSet<XAttrSetFlag> result = 
2135        EnumSet.noneOf(XAttrSetFlag.class);
2136    if ((flag & XAttrSetFlagProto.XATTR_CREATE_VALUE) == 
2137        XAttrSetFlagProto.XATTR_CREATE_VALUE) {
2138      result.add(XAttrSetFlag.CREATE);
2139    }
2140    if ((flag & XAttrSetFlagProto.XATTR_REPLACE_VALUE) == 
2141        XAttrSetFlagProto.XATTR_REPLACE_VALUE) {
2142      result.add(XAttrSetFlag.REPLACE);
2143    }
2144    return result;
2145  }
2146  
2147  public static XAttr convertXAttr(XAttrProto a) {
2148    XAttr.Builder builder = new XAttr.Builder();
2149    builder.setNameSpace(convert(a.getNamespace()));
2150    if (a.hasName()) {
2151      builder.setName(a.getName());
2152    }
2153    if (a.hasValue()) {
2154      builder.setValue(a.getValue().toByteArray());
2155    }
2156    return builder.build();
2157  }
2158  
2159  public static List<XAttr> convertXAttrs(List<XAttrProto> xAttrSpec) {
2160    ArrayList<XAttr> xAttrs = Lists.newArrayListWithCapacity(xAttrSpec.size());
2161    for (XAttrProto a : xAttrSpec) {
2162      XAttr.Builder builder = new XAttr.Builder();
2163      builder.setNameSpace(convert(a.getNamespace()));
2164      if (a.hasName()) {
2165        builder.setName(a.getName());
2166      }
2167      if (a.hasValue()) {
2168        builder.setValue(a.getValue().toByteArray());
2169      }
2170      xAttrs.add(builder.build());
2171    }
2172    return xAttrs;
2173  }
2174  
2175  public static List<XAttr> convert(GetXAttrsResponseProto a) {
2176    List<XAttrProto> xAttrs = a.getXAttrsList();
2177    return convertXAttrs(xAttrs);
2178  }
2179
2180  public static GetXAttrsResponseProto convertXAttrsResponse(
2181      List<XAttr> xAttrs) {
2182    GetXAttrsResponseProto.Builder builder = GetXAttrsResponseProto
2183        .newBuilder();
2184    if (xAttrs != null) {
2185      builder.addAllXAttrs(convertXAttrProto(xAttrs));
2186    }
2187    return builder.build();
2188  }
2189
2190  public static List<XAttr> convert(ListXAttrsResponseProto a) {
2191    final List<XAttrProto> xAttrs = a.getXAttrsList();
2192    return convertXAttrs(xAttrs);
2193  }
2194
2195  public static ListXAttrsResponseProto convertListXAttrsResponse(
2196    List<XAttr> names) {
2197    ListXAttrsResponseProto.Builder builder =
2198      ListXAttrsResponseProto.newBuilder();
2199    if (names != null) {
2200      builder.addAllXAttrs(convertXAttrProto(names));
2201    }
2202    return builder.build();
2203  }
2204
2205  public static ShortCircuitShmSlotProto convert(SlotId slotId) {
2206    return ShortCircuitShmSlotProto.newBuilder().
2207        setShmId(convert(slotId.getShmId())).
2208        setSlotIdx(slotId.getSlotIdx()).
2209        build();
2210  }
2211
2212  public static ShortCircuitShmIdProto convert(ShmId shmId) {
2213    return ShortCircuitShmIdProto.newBuilder().
2214        setHi(shmId.getHi()).
2215        setLo(shmId.getLo()).
2216        build();
2217
2218  }
2219
2220  public static SlotId convert(ShortCircuitShmSlotProto slotId) {
2221    return new SlotId(PBHelper.convert(slotId.getShmId()),
2222        slotId.getSlotIdx());
2223  }
2224
2225  public static ShmId convert(ShortCircuitShmIdProto shmId) {
2226    return new ShmId(shmId.getHi(), shmId.getLo());
2227  }
2228}
2229