View Javadoc

1   /**
2    * Licensed to the Apache Software Foundation (ASF) under one
3    * or more contributor license agreements.  See the NOTICE file
4    * distributed with this work for additional information
5    * regarding copyright ownership.  The ASF licenses this file
6    * to you under the Apache License, Version 2.0 (the
7    * "License"); you may not use this file except in compliance
8    * with the License.  You may obtain a copy of the License at
9    *
10   *     http://www.apache.org/licenses/LICENSE-2.0
11   *
12   * Unless required by applicable law or agreed to in writing, software
13   * distributed under the License is distributed on an "AS IS" BASIS,
14   * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
15   * See the License for the specific language governing permissions and
16   * limitations under the License.
17   */
18  package org.apache.hadoop.hbase.protobuf;
19  
20  
21  import static org.apache.hadoop.hbase.protobuf.generated.HBaseProtos.RegionSpecifier.RegionSpecifierType.REGION_NAME;
22  
23  import java.io.ByteArrayOutputStream;
24  import java.io.IOException;
25  import java.lang.reflect.Constructor;
26  import java.lang.reflect.InvocationTargetException;
27  import java.lang.reflect.Method;
28  import java.lang.reflect.ParameterizedType;
29  import java.lang.reflect.Type;
30  import java.nio.ByteBuffer;
31  import java.util.ArrayList;
32  import java.util.Collection;
33  import java.util.HashMap;
34  import java.util.List;
35  import java.util.Map;
36  import java.util.Map.Entry;
37  import java.util.NavigableSet;
38  import java.util.concurrent.TimeUnit;
39  
40  import org.apache.hadoop.conf.Configuration;
41  import org.apache.hadoop.fs.Path;
42  import org.apache.hadoop.hbase.Cell;
43  import org.apache.hadoop.hbase.CellScanner;
44  import org.apache.hadoop.hbase.CellUtil;
45  import org.apache.hadoop.hbase.DoNotRetryIOException;
46  import org.apache.hadoop.hbase.HBaseConfiguration;
47  import org.apache.hadoop.hbase.HConstants;
48  import org.apache.hadoop.hbase.HRegionInfo;
49  import org.apache.hadoop.hbase.HTableDescriptor;
50  import org.apache.hadoop.hbase.KeyValue;
51  import org.apache.hadoop.hbase.NamespaceDescriptor;
52  import org.apache.hadoop.hbase.ServerName;
53  import org.apache.hadoop.hbase.TableName;
54  import org.apache.hadoop.hbase.Tag;
55  import org.apache.hadoop.hbase.client.Append;
56  import org.apache.hadoop.hbase.client.Consistency;
57  import org.apache.hadoop.hbase.client.Delete;
58  import org.apache.hadoop.hbase.client.Durability;
59  import org.apache.hadoop.hbase.client.Get;
60  import org.apache.hadoop.hbase.client.Increment;
61  import org.apache.hadoop.hbase.client.Mutation;
62  import org.apache.hadoop.hbase.client.Put;
63  import org.apache.hadoop.hbase.client.Result;
64  import org.apache.hadoop.hbase.client.Scan;
65  import org.apache.hadoop.hbase.client.metrics.ScanMetrics;
66  import org.apache.hadoop.hbase.exceptions.DeserializationException;
67  import org.apache.hadoop.hbase.filter.ByteArrayComparable;
68  import org.apache.hadoop.hbase.filter.Filter;
69  import org.apache.hadoop.hbase.io.TimeRange;
70  import org.apache.hadoop.hbase.protobuf.generated.AccessControlProtos;
71  import org.apache.hadoop.hbase.protobuf.generated.AccessControlProtos.AccessControlService;
72  import org.apache.hadoop.hbase.protobuf.generated.AdminProtos.AdminService;
73  import org.apache.hadoop.hbase.protobuf.generated.AdminProtos.CloseRegionRequest;
74  import org.apache.hadoop.hbase.protobuf.generated.AdminProtos.CloseRegionResponse;
75  import org.apache.hadoop.hbase.protobuf.generated.AdminProtos.GetOnlineRegionRequest;
76  import org.apache.hadoop.hbase.protobuf.generated.AdminProtos.GetOnlineRegionResponse;
77  import org.apache.hadoop.hbase.protobuf.generated.AdminProtos.GetRegionInfoRequest;
78  import org.apache.hadoop.hbase.protobuf.generated.AdminProtos.GetRegionInfoResponse;
79  import org.apache.hadoop.hbase.protobuf.generated.AdminProtos.GetServerInfoRequest;
80  import org.apache.hadoop.hbase.protobuf.generated.AdminProtos.GetServerInfoResponse;
81  import org.apache.hadoop.hbase.protobuf.generated.AdminProtos.GetStoreFileRequest;
82  import org.apache.hadoop.hbase.protobuf.generated.AdminProtos.GetStoreFileResponse;
83  import org.apache.hadoop.hbase.protobuf.generated.AdminProtos.MergeRegionsRequest;
84  import org.apache.hadoop.hbase.protobuf.generated.AdminProtos.OpenRegionRequest;
85  import org.apache.hadoop.hbase.protobuf.generated.AdminProtos.ServerInfo;
86  import org.apache.hadoop.hbase.protobuf.generated.AdminProtos.SplitRegionRequest;
87  import org.apache.hadoop.hbase.protobuf.generated.AuthenticationProtos;
88  import org.apache.hadoop.hbase.protobuf.generated.CellProtos;
89  import org.apache.hadoop.hbase.protobuf.generated.ClientProtos;
90  import org.apache.hadoop.hbase.protobuf.generated.ClientProtos.BulkLoadHFileRequest;
91  import org.apache.hadoop.hbase.protobuf.generated.ClientProtos.BulkLoadHFileResponse;
92  import org.apache.hadoop.hbase.protobuf.generated.ClientProtos.ClientService;
93  import org.apache.hadoop.hbase.protobuf.generated.ClientProtos.Column;
94  import org.apache.hadoop.hbase.protobuf.generated.ClientProtos.CoprocessorServiceCall;
95  import org.apache.hadoop.hbase.protobuf.generated.ClientProtos.CoprocessorServiceRequest;
96  import org.apache.hadoop.hbase.protobuf.generated.ClientProtos.CoprocessorServiceResponse;
97  import org.apache.hadoop.hbase.protobuf.generated.ClientProtos.GetRequest;
98  import org.apache.hadoop.hbase.protobuf.generated.ClientProtos.GetResponse;
99  import org.apache.hadoop.hbase.protobuf.generated.ClientProtos.MutationProto;
100 import org.apache.hadoop.hbase.protobuf.generated.ClientProtos.MutationProto.ColumnValue;
101 import org.apache.hadoop.hbase.protobuf.generated.ClientProtos.MutationProto.ColumnValue.QualifierValue;
102 import org.apache.hadoop.hbase.protobuf.generated.ClientProtos.MutationProto.DeleteType;
103 import org.apache.hadoop.hbase.protobuf.generated.ClientProtos.MutationProto.MutationType;
104 import org.apache.hadoop.hbase.protobuf.generated.ClientProtos.ScanRequest;
105 import org.apache.hadoop.hbase.protobuf.generated.ClusterStatusProtos.RegionLoad;
106 import org.apache.hadoop.hbase.protobuf.generated.ComparatorProtos;
107 import org.apache.hadoop.hbase.protobuf.generated.FilterProtos;
108 import org.apache.hadoop.hbase.protobuf.generated.HBaseProtos;
109 import org.apache.hadoop.hbase.protobuf.generated.HBaseProtos.NameBytesPair;
110 import org.apache.hadoop.hbase.protobuf.generated.HBaseProtos.RegionInfo;
111 import org.apache.hadoop.hbase.protobuf.generated.HBaseProtos.RegionSpecifier;
112 import org.apache.hadoop.hbase.protobuf.generated.HBaseProtos.RegionSpecifier.RegionSpecifierType;
113 import org.apache.hadoop.hbase.protobuf.generated.MapReduceProtos;
114 import org.apache.hadoop.hbase.protobuf.generated.MasterProtos.CreateTableRequest;
115 import org.apache.hadoop.hbase.protobuf.generated.MasterProtos.GetTableDescriptorsResponse;
116 import org.apache.hadoop.hbase.protobuf.generated.MasterProtos.MasterService;
117 import org.apache.hadoop.hbase.protobuf.generated.QuotaProtos;
118 import org.apache.hadoop.hbase.protobuf.generated.RegionServerStatusProtos.RegionServerReportRequest;
119 import org.apache.hadoop.hbase.protobuf.generated.RegionServerStatusProtos.RegionServerStartupRequest;
120 import org.apache.hadoop.hbase.protobuf.generated.WALProtos;
121 import org.apache.hadoop.hbase.protobuf.generated.WALProtos.CompactionDescriptor;
122 import org.apache.hadoop.hbase.protobuf.generated.WALProtos.FlushDescriptor;
123 import org.apache.hadoop.hbase.protobuf.generated.WALProtos.FlushDescriptor.FlushAction;
124 import org.apache.hadoop.hbase.protobuf.generated.WALProtos.RegionEventDescriptor;
125 import org.apache.hadoop.hbase.protobuf.generated.WALProtos.RegionEventDescriptor.EventType;
126 import org.apache.hadoop.hbase.quotas.QuotaScope;
127 import org.apache.hadoop.hbase.quotas.QuotaType;
128 import org.apache.hadoop.hbase.quotas.ThrottleType;
129 import org.apache.hadoop.hbase.security.access.Permission;
130 import org.apache.hadoop.hbase.security.access.TablePermission;
131 import org.apache.hadoop.hbase.security.access.UserPermission;
132 import org.apache.hadoop.hbase.security.token.AuthenticationTokenIdentifier;
133 import org.apache.hadoop.hbase.security.visibility.Authorizations;
134 import org.apache.hadoop.hbase.security.visibility.CellVisibility;
135 import org.apache.hadoop.hbase.util.ByteStringer;
136 import org.apache.hadoop.hbase.util.Bytes;
137 import org.apache.hadoop.hbase.util.DynamicClassLoader;
138 import org.apache.hadoop.hbase.util.ExceptionUtil;
139 import org.apache.hadoop.hbase.util.Methods;
140 import org.apache.hadoop.hbase.util.Pair;
141 import org.apache.hadoop.io.Text;
142 import org.apache.hadoop.ipc.RemoteException;
143 import org.apache.hadoop.security.token.Token;
144 
145 import com.google.common.collect.ArrayListMultimap;
146 import com.google.common.collect.ListMultimap;
147 import com.google.common.collect.Lists;
148 import com.google.protobuf.ByteString;
149 import com.google.protobuf.InvalidProtocolBufferException;
150 import com.google.protobuf.Message;
151 import com.google.protobuf.Parser;
152 import com.google.protobuf.RpcChannel;
153 import com.google.protobuf.Service;
154 import com.google.protobuf.ServiceException;
155 import com.google.protobuf.TextFormat;
156 
157 /**
158  * Protobufs utility.
159  */
160 @edu.umd.cs.findbugs.annotations.SuppressWarnings(value="DP_CREATE_CLASSLOADER_INSIDE_DO_PRIVILEGED",
161   justification="None. Address sometime.")
162 public final class ProtobufUtil {
163 
164   private ProtobufUtil() {
165   }
166 
167   /**
168    * Primitive type to class mapping.
169    */
170   private final static Map<String, Class<?>>
171     PRIMITIVES = new HashMap<String, Class<?>>();
172 
173 
174   /**
175    * Many results are simple: no cell, exists true or false. To save on object creations,
176    *  we reuse them across calls.
177    */
178   private final static Cell[] EMPTY_CELL_ARRAY = new Cell[]{};
179   private final static Result EMPTY_RESULT = Result.create(EMPTY_CELL_ARRAY);
180   private final static Result EMPTY_RESULT_EXISTS_TRUE = Result.create(null, true);
181   private final static Result EMPTY_RESULT_EXISTS_FALSE = Result.create(null, false);
182   private final static Result EMPTY_RESULT_STALE = Result.create(EMPTY_CELL_ARRAY, null, true);
183   private final static Result EMPTY_RESULT_EXISTS_TRUE_STALE
184     = Result.create((Cell[])null, true, true);
185   private final static Result EMPTY_RESULT_EXISTS_FALSE_STALE
186     = Result.create((Cell[])null, false, true);
187 
188   private final static ClientProtos.Result EMPTY_RESULT_PB;
189   private final static ClientProtos.Result EMPTY_RESULT_PB_EXISTS_TRUE;
190   private final static ClientProtos.Result EMPTY_RESULT_PB_EXISTS_FALSE;
191   private final static ClientProtos.Result EMPTY_RESULT_PB_STALE;
192   private final static ClientProtos.Result EMPTY_RESULT_PB_EXISTS_TRUE_STALE;
193   private final static ClientProtos.Result EMPTY_RESULT_PB_EXISTS_FALSE_STALE;
194 
195 
196   static {
197     ClientProtos.Result.Builder builder = ClientProtos.Result.newBuilder();
198 
199     builder.setExists(true);
200     builder.setAssociatedCellCount(0);
201     EMPTY_RESULT_PB_EXISTS_TRUE =  builder.build();
202 
203     builder.setStale(true);
204     EMPTY_RESULT_PB_EXISTS_TRUE_STALE = builder.build();
205     builder.clear();
206 
207     builder.setExists(false);
208     builder.setAssociatedCellCount(0);
209     EMPTY_RESULT_PB_EXISTS_FALSE =  builder.build();
210     builder.setStale(true);
211     EMPTY_RESULT_PB_EXISTS_FALSE_STALE = builder.build();
212 
213     builder.clear();
214     builder.setAssociatedCellCount(0);
215     EMPTY_RESULT_PB =  builder.build();
216     builder.setStale(true);
217     EMPTY_RESULT_PB_STALE = builder.build();
218   }
219 
220   /**
221    * Dynamic class loader to load filter/comparators
222    */
223   private final static ClassLoader CLASS_LOADER;
224 
225   static {
226     ClassLoader parent = ProtobufUtil.class.getClassLoader();
227     Configuration conf = HBaseConfiguration.create();
228     CLASS_LOADER = new DynamicClassLoader(conf, parent);
229 
230     PRIMITIVES.put(Boolean.TYPE.getName(), Boolean.TYPE);
231     PRIMITIVES.put(Byte.TYPE.getName(), Byte.TYPE);
232     PRIMITIVES.put(Character.TYPE.getName(), Character.TYPE);
233     PRIMITIVES.put(Short.TYPE.getName(), Short.TYPE);
234     PRIMITIVES.put(Integer.TYPE.getName(), Integer.TYPE);
235     PRIMITIVES.put(Long.TYPE.getName(), Long.TYPE);
236     PRIMITIVES.put(Float.TYPE.getName(), Float.TYPE);
237     PRIMITIVES.put(Double.TYPE.getName(), Double.TYPE);
238     PRIMITIVES.put(Void.TYPE.getName(), Void.TYPE);
239   }
240 
241   /**
242    * Magic we put ahead of a serialized protobuf message.
243    * For example, all znode content is protobuf messages with the below magic
244    * for preamble.
245    */
246   public static final byte [] PB_MAGIC = new byte [] {'P', 'B', 'U', 'F'};
247   private static final String PB_MAGIC_STR = Bytes.toString(PB_MAGIC);
248 
249   /**
250    * Prepend the passed bytes with four bytes of magic, {@link #PB_MAGIC}, to flag what
251    * follows as a protobuf in hbase.  Prepend these bytes to all content written to znodes, etc.
252    * @param bytes Bytes to decorate
253    * @return The passed <code>bytes</codes> with magic prepended (Creates a new
254    * byte array that is <code>bytes.length</code> plus {@link #PB_MAGIC}.length.
255    */
256   public static byte [] prependPBMagic(final byte [] bytes) {
257     return Bytes.add(PB_MAGIC, bytes);
258   }
259 
260   /**
261    * @param bytes Bytes to check.
262    * @return True if passed <code>bytes</code> has {@link #PB_MAGIC} for a prefix.
263    */
264   public static boolean isPBMagicPrefix(final byte [] bytes) {
265     if (bytes == null) return false;
266     return isPBMagicPrefix(bytes, 0, bytes.length);
267   }
268 
269   /**
270    * @param bytes Bytes to check.
271    * @return True if passed <code>bytes</code> has {@link #PB_MAGIC} for a prefix.
272    */
273   public static boolean isPBMagicPrefix(final byte [] bytes, int offset, int len) {
274     if (bytes == null || len < PB_MAGIC.length) return false;
275     return Bytes.compareTo(PB_MAGIC, 0, PB_MAGIC.length, bytes, offset, PB_MAGIC.length) == 0;
276   }
277 
278   /**
279    * @param bytes
280    * @throws DeserializationException if we are missing the pb magic prefix
281    */
282   public static void expectPBMagicPrefix(final byte [] bytes) throws DeserializationException {
283     if (!isPBMagicPrefix(bytes)) {
284       throw new DeserializationException("Missing pb magic " + PB_MAGIC_STR + " prefix");
285     }
286   }
287 
288   /**
289    * @return Length of {@link #PB_MAGIC}
290    */
291   public static int lengthOfPBMagic() {
292     return PB_MAGIC.length;
293   }
294 
295   /**
296    * Return the IOException thrown by the remote server wrapped in
297    * ServiceException as cause.
298    *
299    * @param se ServiceException that wraps IO exception thrown by the server
300    * @return Exception wrapped in ServiceException or
301    *   a new IOException that wraps the unexpected ServiceException.
302    */
303   public static IOException getRemoteException(ServiceException se) {
304     Throwable e = se.getCause();
305     if (e == null) {
306       return new IOException(se);
307     }
308     if (ExceptionUtil.isInterrupt(e)) {
309       return ExceptionUtil.asInterrupt(e);
310     }
311     if (e instanceof RemoteException) {
312       e = ((RemoteException) e).unwrapRemoteException();
313     }
314     return e instanceof IOException ? (IOException) e : new IOException(se);
315   }
316 
317   /**
318    * Convert a ServerName to a protocol buffer ServerName
319    *
320    * @param serverName the ServerName to convert
321    * @return the converted protocol buffer ServerName
322    * @see #toServerName(org.apache.hadoop.hbase.protobuf.generated.HBaseProtos.ServerName)
323    */
324   public static HBaseProtos.ServerName
325       toServerName(final ServerName serverName) {
326     if (serverName == null) return null;
327     HBaseProtos.ServerName.Builder builder =
328       HBaseProtos.ServerName.newBuilder();
329     builder.setHostName(serverName.getHostname());
330     if (serverName.getPort() >= 0) {
331       builder.setPort(serverName.getPort());
332     }
333     if (serverName.getStartcode() >= 0) {
334       builder.setStartCode(serverName.getStartcode());
335     }
336     return builder.build();
337   }
338 
339   /**
340    * Convert a protocol buffer ServerName to a ServerName
341    *
342    * @param proto the protocol buffer ServerName to convert
343    * @return the converted ServerName
344    */
345   public static ServerName toServerName(final HBaseProtos.ServerName proto) {
346     if (proto == null) return null;
347     String hostName = proto.getHostName();
348     long startCode = -1;
349     int port = -1;
350     if (proto.hasPort()) {
351       port = proto.getPort();
352     }
353     if (proto.hasStartCode()) {
354       startCode = proto.getStartCode();
355     }
356     return ServerName.valueOf(hostName, port, startCode);
357   }
358 
359   /**
360    * Get HTableDescriptor[] from GetTableDescriptorsResponse protobuf
361    *
362    * @param proto the GetTableDescriptorsResponse
363    * @return HTableDescriptor[]
364    */
365   public static HTableDescriptor[] getHTableDescriptorArray(GetTableDescriptorsResponse proto) {
366     if (proto == null) return null;
367 
368     HTableDescriptor[] ret = new HTableDescriptor[proto.getTableSchemaCount()];
369     for (int i = 0; i < proto.getTableSchemaCount(); ++i) {
370       ret[i] = HTableDescriptor.convert(proto.getTableSchema(i));
371     }
372     return ret;
373   }
374 
375   /**
376    * get the split keys in form "byte [][]" from a CreateTableRequest proto
377    *
378    * @param proto the CreateTableRequest
379    * @return the split keys
380    */
381   public static byte [][] getSplitKeysArray(final CreateTableRequest proto) {
382     byte [][] splitKeys = new byte[proto.getSplitKeysCount()][];
383     for (int i = 0; i < proto.getSplitKeysCount(); ++i) {
384       splitKeys[i] = proto.getSplitKeys(i).toByteArray();
385     }
386     return splitKeys;
387   }
388 
389   /**
390    * Convert a protobuf Durability into a client Durability
391    */
392   public static Durability toDurability(
393       final ClientProtos.MutationProto.Durability proto) {
394     switch(proto) {
395     case USE_DEFAULT:
396       return Durability.USE_DEFAULT;
397     case SKIP_WAL:
398       return Durability.SKIP_WAL;
399     case ASYNC_WAL:
400       return Durability.ASYNC_WAL;
401     case SYNC_WAL:
402       return Durability.SYNC_WAL;
403     case FSYNC_WAL:
404       return Durability.FSYNC_WAL;
405     default:
406       return Durability.USE_DEFAULT;
407     }
408   }
409 
410   /**
411    * Convert a client Durability into a protbuf Durability
412    */
413   public static ClientProtos.MutationProto.Durability toDurability(
414       final Durability d) {
415     switch(d) {
416     case USE_DEFAULT:
417       return ClientProtos.MutationProto.Durability.USE_DEFAULT;
418     case SKIP_WAL:
419       return ClientProtos.MutationProto.Durability.SKIP_WAL;
420     case ASYNC_WAL:
421       return ClientProtos.MutationProto.Durability.ASYNC_WAL;
422     case SYNC_WAL:
423       return ClientProtos.MutationProto.Durability.SYNC_WAL;
424     case FSYNC_WAL:
425       return ClientProtos.MutationProto.Durability.FSYNC_WAL;
426     default:
427       return ClientProtos.MutationProto.Durability.USE_DEFAULT;
428     }
429   }
430 
431   /**
432    * Convert a protocol buffer Get to a client Get
433    *
434    * @param proto the protocol buffer Get to convert
435    * @return the converted client Get
436    * @throws IOException
437    */
438   public static Get toGet(
439       final ClientProtos.Get proto) throws IOException {
440     if (proto == null) return null;
441     byte[] row = proto.getRow().toByteArray();
442     Get get = new Get(row);
443     if (proto.hasCacheBlocks()) {
444       get.setCacheBlocks(proto.getCacheBlocks());
445     }
446     if (proto.hasMaxVersions()) {
447       get.setMaxVersions(proto.getMaxVersions());
448     }
449     if (proto.hasStoreLimit()) {
450       get.setMaxResultsPerColumnFamily(proto.getStoreLimit());
451     }
452     if (proto.hasStoreOffset()) {
453       get.setRowOffsetPerColumnFamily(proto.getStoreOffset());
454     }
455     if (proto.hasTimeRange()) {
456       HBaseProtos.TimeRange timeRange = proto.getTimeRange();
457       long minStamp = 0;
458       long maxStamp = Long.MAX_VALUE;
459       if (timeRange.hasFrom()) {
460         minStamp = timeRange.getFrom();
461       }
462       if (timeRange.hasTo()) {
463         maxStamp = timeRange.getTo();
464       }
465       get.setTimeRange(minStamp, maxStamp);
466     }
467     if (proto.hasFilter()) {
468       FilterProtos.Filter filter = proto.getFilter();
469       get.setFilter(ProtobufUtil.toFilter(filter));
470     }
471     for (NameBytesPair attribute: proto.getAttributeList()) {
472       get.setAttribute(attribute.getName(), attribute.getValue().toByteArray());
473     }
474     if (proto.getColumnCount() > 0) {
475       for (Column column: proto.getColumnList()) {
476         byte[] family = column.getFamily().toByteArray();
477         if (column.getQualifierCount() > 0) {
478           for (ByteString qualifier: column.getQualifierList()) {
479             get.addColumn(family, qualifier.toByteArray());
480           }
481         } else {
482           get.addFamily(family);
483         }
484       }
485     }
486     if (proto.hasExistenceOnly() && proto.getExistenceOnly()){
487       get.setCheckExistenceOnly(true);
488     }
489     if (proto.hasClosestRowBefore() && proto.getClosestRowBefore()){
490       get.setClosestRowBefore(true);
491     }
492     if (proto.hasConsistency()) {
493       get.setConsistency(toConsistency(proto.getConsistency()));
494     }
495     return get;
496   }
497 
498   public static Consistency toConsistency(ClientProtos.Consistency consistency) {
499     switch (consistency) {
500       case STRONG : return Consistency.STRONG;
501       case TIMELINE : return Consistency.TIMELINE;
502       default : return Consistency.STRONG;
503     }
504   }
505 
506   public static ClientProtos.Consistency toConsistency(Consistency consistency) {
507     switch (consistency) {
508       case STRONG : return ClientProtos.Consistency.STRONG;
509       case TIMELINE : return ClientProtos.Consistency.TIMELINE;
510       default : return ClientProtos.Consistency.STRONG;
511     }
512   }
513 
514   /**
515    * Convert a protocol buffer Mutate to a Put.
516    *
517    * @param proto The protocol buffer MutationProto to convert
518    * @return A client Put.
519    * @throws IOException
520    */
521   public static Put toPut(final MutationProto proto)
522   throws IOException {
523     return toPut(proto, null);
524   }
525 
526   /**
527    * Convert a protocol buffer Mutate to a Put.
528    *
529    * @param proto The protocol buffer MutationProto to convert
530    * @param cellScanner If non-null, the Cell data that goes with this proto.
531    * @return A client Put.
532    * @throws IOException
533    */
534   public static Put toPut(final MutationProto proto, final CellScanner cellScanner)
535   throws IOException {
536     // TODO: Server-side at least why do we convert back to the Client types?  Why not just pb it?
537     MutationType type = proto.getMutateType();
538     assert type == MutationType.PUT: type.name();
539     long timestamp = proto.hasTimestamp()? proto.getTimestamp(): HConstants.LATEST_TIMESTAMP;
540     Put put = null;
541     int cellCount = proto.hasAssociatedCellCount()? proto.getAssociatedCellCount(): 0;
542     if (cellCount > 0) {
543       // The proto has metadata only and the data is separate to be found in the cellScanner.
544       if (cellScanner == null) {
545         throw new DoNotRetryIOException("Cell count of " + cellCount + " but no cellScanner: " +
546             toShortString(proto));
547       }
548       for (int i = 0; i < cellCount; i++) {
549         if (!cellScanner.advance()) {
550           throw new DoNotRetryIOException("Cell count of " + cellCount + " but at index " + i +
551             " no cell returned: " + toShortString(proto));
552         }
553         Cell cell = cellScanner.current();
554         if (put == null) {
555           put = new Put(cell.getRowArray(), cell.getRowOffset(), cell.getRowLength(), timestamp);
556         }
557         put.add(cell);
558       }
559     } else {
560       if (proto.hasRow()) {
561         put = new Put(proto.getRow().asReadOnlyByteBuffer(), timestamp);
562       } else {
563         throw new IllegalArgumentException("row cannot be null");
564       }
565       // The proto has the metadata and the data itself
566       for (ColumnValue column: proto.getColumnValueList()) {
567         byte[] family = column.getFamily().toByteArray();
568         for (QualifierValue qv: column.getQualifierValueList()) {
569           if (!qv.hasValue()) {
570             throw new DoNotRetryIOException(
571                 "Missing required field: qualifier value");
572           }
573           ByteBuffer qualifier =
574               qv.hasQualifier() ? qv.getQualifier().asReadOnlyByteBuffer() : null;
575           ByteBuffer value =
576               qv.hasValue() ? qv.getValue().asReadOnlyByteBuffer() : null;
577           long ts = timestamp;
578           if (qv.hasTimestamp()) {
579             ts = qv.getTimestamp();
580           }
581           byte[] tags;
582           if (qv.hasTags()) {
583             tags = qv.getTags().toByteArray();
584             Object[] array = Tag.asList(tags, 0, (short)tags.length).toArray();
585             Tag[] tagArray = new Tag[array.length];
586             for(int i = 0; i< array.length; i++) {
587               tagArray[i] = (Tag)array[i];
588             }
589             if(qv.hasDeleteType()) {
590               byte[] qual = qv.hasQualifier() ? qv.getQualifier().toByteArray() : null;
591               put.add(new KeyValue(proto.getRow().toByteArray(), family, qual, ts,
592                   fromDeleteType(qv.getDeleteType()), null, tags));
593             } else {
594               put.addImmutable(family, qualifier, ts, value, tagArray);
595             }
596           } else {
597             if(qv.hasDeleteType()) {
598               byte[] qual = qv.hasQualifier() ? qv.getQualifier().toByteArray() : null;
599               put.add(new KeyValue(proto.getRow().toByteArray(), family, qual, ts,
600                   fromDeleteType(qv.getDeleteType())));
601             } else{
602               put.addImmutable(family, qualifier, ts, value);
603             }
604           }
605         }
606       }
607     }
608     put.setDurability(toDurability(proto.getDurability()));
609     for (NameBytesPair attribute: proto.getAttributeList()) {
610       put.setAttribute(attribute.getName(), attribute.getValue().toByteArray());
611     }
612     return put;
613   }
614 
615   /**
616    * Convert a protocol buffer Mutate to a Delete
617    *
618    * @param proto the protocol buffer Mutate to convert
619    * @return the converted client Delete
620    * @throws IOException
621    */
622   public static Delete toDelete(final MutationProto proto)
623   throws IOException {
624     return toDelete(proto, null);
625   }
626 
627   /**
628    * Convert a protocol buffer Mutate to a Delete
629    *
630    * @param proto the protocol buffer Mutate to convert
631    * @param cellScanner if non-null, the data that goes with this delete.
632    * @return the converted client Delete
633    * @throws IOException
634    */
635   public static Delete toDelete(final MutationProto proto, final CellScanner cellScanner)
636   throws IOException {
637     MutationType type = proto.getMutateType();
638     assert type == MutationType.DELETE : type.name();
639     byte [] row = proto.hasRow()? proto.getRow().toByteArray(): null;
640     long timestamp = HConstants.LATEST_TIMESTAMP;
641     if (proto.hasTimestamp()) {
642       timestamp = proto.getTimestamp();
643     }
644     Delete delete = null;
645     int cellCount = proto.hasAssociatedCellCount()? proto.getAssociatedCellCount(): 0;
646     if (cellCount > 0) {
647       // The proto has metadata only and the data is separate to be found in the cellScanner.
648       if (cellScanner == null) {
649         // TextFormat should be fine for a Delete since it carries no data, just coordinates.
650         throw new DoNotRetryIOException("Cell count of " + cellCount + " but no cellScanner: " +
651           TextFormat.shortDebugString(proto));
652       }
653       for (int i = 0; i < cellCount; i++) {
654         if (!cellScanner.advance()) {
655           // TextFormat should be fine for a Delete since it carries no data, just coordinates.
656           throw new DoNotRetryIOException("Cell count of " + cellCount + " but at index " + i +
657             " no cell returned: " + TextFormat.shortDebugString(proto));
658         }
659         Cell cell = cellScanner.current();
660         if (delete == null) {
661           delete =
662             new Delete(cell.getRowArray(), cell.getRowOffset(), cell.getRowLength(), timestamp);
663         }
664         delete.addDeleteMarker(cell);
665       }
666     } else {
667       delete = new Delete(row, timestamp);
668       for (ColumnValue column: proto.getColumnValueList()) {
669         byte[] family = column.getFamily().toByteArray();
670         for (QualifierValue qv: column.getQualifierValueList()) {
671           DeleteType deleteType = qv.getDeleteType();
672           byte[] qualifier = null;
673           if (qv.hasQualifier()) {
674             qualifier = qv.getQualifier().toByteArray();
675           }
676           long ts = HConstants.LATEST_TIMESTAMP;
677           if (qv.hasTimestamp()) {
678             ts = qv.getTimestamp();
679           }
680           if (deleteType == DeleteType.DELETE_ONE_VERSION) {
681             delete.deleteColumn(family, qualifier, ts);
682           } else if (deleteType == DeleteType.DELETE_MULTIPLE_VERSIONS) {
683             delete.deleteColumns(family, qualifier, ts);
684           } else if (deleteType == DeleteType.DELETE_FAMILY_VERSION) {
685             delete.deleteFamilyVersion(family, ts);
686           } else {
687             delete.deleteFamily(family, ts);
688           }
689         }
690       }
691     }
692     delete.setDurability(toDurability(proto.getDurability()));
693     for (NameBytesPair attribute: proto.getAttributeList()) {
694       delete.setAttribute(attribute.getName(), attribute.getValue().toByteArray());
695     }
696     return delete;
697   }
698 
699   /**
700    * Convert a protocol buffer Mutate to an Append
701    * @param cellScanner
702    * @param proto the protocol buffer Mutate to convert
703    * @return the converted client Append
704    * @throws IOException
705    */
706   public static Append toAppend(final MutationProto proto, final CellScanner cellScanner)
707   throws IOException {
708     MutationType type = proto.getMutateType();
709     assert type == MutationType.APPEND : type.name();
710     byte [] row = proto.hasRow()? proto.getRow().toByteArray(): null;
711     Append append = null;
712     int cellCount = proto.hasAssociatedCellCount()? proto.getAssociatedCellCount(): 0;
713     if (cellCount > 0) {
714       // The proto has metadata only and the data is separate to be found in the cellScanner.
715       if (cellScanner == null) {
716         throw new DoNotRetryIOException("Cell count of " + cellCount + " but no cellScanner: " +
717           toShortString(proto));
718       }
719       for (int i = 0; i < cellCount; i++) {
720         if (!cellScanner.advance()) {
721           throw new DoNotRetryIOException("Cell count of " + cellCount + " but at index " + i +
722             " no cell returned: " + toShortString(proto));
723         }
724         Cell cell = cellScanner.current();
725         if (append == null) {
726           append = new Append(cell.getRowArray(), cell.getRowOffset(), cell.getRowLength());
727         }
728         append.add(cell);
729       }
730     } else {
731       append = new Append(row);
732       for (ColumnValue column: proto.getColumnValueList()) {
733         byte[] family = column.getFamily().toByteArray();
734         for (QualifierValue qv: column.getQualifierValueList()) {
735           byte[] qualifier = qv.getQualifier().toByteArray();
736           if (!qv.hasValue()) {
737             throw new DoNotRetryIOException(
738               "Missing required field: qualifier value");
739           }
740           byte[] value = qv.getValue().toByteArray();
741           byte[] tags = null;
742           if (qv.hasTags()) {
743             tags = qv.getTags().toByteArray();
744           }
745           append.add(CellUtil.createCell(row, family, qualifier, qv.getTimestamp(),
746               KeyValue.Type.Put, value, tags));
747         }
748       }
749     }
750     append.setDurability(toDurability(proto.getDurability()));
751     for (NameBytesPair attribute: proto.getAttributeList()) {
752       append.setAttribute(attribute.getName(), attribute.getValue().toByteArray());
753     }
754     return append;
755   }
756 
757   /**
758    * Convert a MutateRequest to Mutation
759    *
760    * @param proto the protocol buffer Mutate to convert
761    * @return the converted Mutation
762    * @throws IOException
763    */
764   public static Mutation toMutation(final MutationProto proto) throws IOException {
765     MutationType type = proto.getMutateType();
766     if (type == MutationType.APPEND) {
767       return toAppend(proto, null);
768     }
769     if (type == MutationType.DELETE) {
770       return toDelete(proto, null);
771     }
772     if (type == MutationType.PUT) {
773       return toPut(proto, null);
774     }
775     throw new IOException("Unknown mutation type " + type);
776   }
777 
778   /**
779    * Convert a protocol buffer Mutate to an Increment
780    *
781    * @param proto the protocol buffer Mutate to convert
782    * @return the converted client Increment
783    * @throws IOException
784    */
785   public static Increment toIncrement(final MutationProto proto, final CellScanner cellScanner)
786   throws IOException {
787     MutationType type = proto.getMutateType();
788     assert type == MutationType.INCREMENT : type.name();
789     byte [] row = proto.hasRow()? proto.getRow().toByteArray(): null;
790     Increment increment = null;
791     int cellCount = proto.hasAssociatedCellCount()? proto.getAssociatedCellCount(): 0;
792     if (cellCount > 0) {
793       // The proto has metadata only and the data is separate to be found in the cellScanner.
794       if (cellScanner == null) {
795         throw new DoNotRetryIOException("Cell count of " + cellCount + " but no cellScanner: " +
796           TextFormat.shortDebugString(proto));
797       }
798       for (int i = 0; i < cellCount; i++) {
799         if (!cellScanner.advance()) {
800           throw new DoNotRetryIOException("Cell count of " + cellCount + " but at index " + i +
801             " no cell returned: " + TextFormat.shortDebugString(proto));
802         }
803         Cell cell = cellScanner.current();
804         if (increment == null) {
805           increment = new Increment(cell.getRowArray(), cell.getRowOffset(), cell.getRowLength());
806         }
807         increment.add(cell);
808       }
809     } else {
810       increment = new Increment(row);
811       for (ColumnValue column: proto.getColumnValueList()) {
812         byte[] family = column.getFamily().toByteArray();
813         for (QualifierValue qv: column.getQualifierValueList()) {
814           byte[] qualifier = qv.getQualifier().toByteArray();
815           if (!qv.hasValue()) {
816             throw new DoNotRetryIOException("Missing required field: qualifier value");
817           }
818           byte[] value = qv.getValue().toByteArray();
819           byte[] tags = null;
820           if (qv.hasTags()) {
821             tags = qv.getTags().toByteArray();
822           }
823           increment.add(CellUtil.createCell(row, family, qualifier, qv.getTimestamp(),
824               KeyValue.Type.Put, value, tags));
825         }
826       }
827     }
828     if (proto.hasTimeRange()) {
829       HBaseProtos.TimeRange timeRange = proto.getTimeRange();
830       long minStamp = 0;
831       long maxStamp = Long.MAX_VALUE;
832       if (timeRange.hasFrom()) {
833         minStamp = timeRange.getFrom();
834       }
835       if (timeRange.hasTo()) {
836         maxStamp = timeRange.getTo();
837       }
838       increment.setTimeRange(minStamp, maxStamp);
839     }
840     increment.setDurability(toDurability(proto.getDurability()));
841     for (NameBytesPair attribute : proto.getAttributeList()) {
842       increment.setAttribute(attribute.getName(), attribute.getValue().toByteArray());
843     }
844     return increment;
845   }
846 
847   /**
848    * Convert a client Scan to a protocol buffer Scan
849    *
850    * @param scan the client Scan to convert
851    * @return the converted protocol buffer Scan
852    * @throws IOException
853    */
854   public static ClientProtos.Scan toScan(
855       final Scan scan) throws IOException {
856     ClientProtos.Scan.Builder scanBuilder =
857       ClientProtos.Scan.newBuilder();
858     scanBuilder.setCacheBlocks(scan.getCacheBlocks());
859     if (scan.getBatch() > 0) {
860       scanBuilder.setBatchSize(scan.getBatch());
861     }
862     if (scan.getMaxResultSize() > 0) {
863       scanBuilder.setMaxResultSize(scan.getMaxResultSize());
864     }
865     if (scan.isSmall()) {
866       scanBuilder.setSmall(scan.isSmall());
867     }
868     Boolean loadColumnFamiliesOnDemand = scan.getLoadColumnFamiliesOnDemandValue();
869     if (loadColumnFamiliesOnDemand != null) {
870       scanBuilder.setLoadColumnFamiliesOnDemand(loadColumnFamiliesOnDemand.booleanValue());
871     }
872     scanBuilder.setMaxVersions(scan.getMaxVersions());
873     TimeRange timeRange = scan.getTimeRange();
874     if (!timeRange.isAllTime()) {
875       HBaseProtos.TimeRange.Builder timeRangeBuilder =
876         HBaseProtos.TimeRange.newBuilder();
877       timeRangeBuilder.setFrom(timeRange.getMin());
878       timeRangeBuilder.setTo(timeRange.getMax());
879       scanBuilder.setTimeRange(timeRangeBuilder.build());
880     }
881     Map<String, byte[]> attributes = scan.getAttributesMap();
882     if (!attributes.isEmpty()) {
883       NameBytesPair.Builder attributeBuilder = NameBytesPair.newBuilder();
884       for (Map.Entry<String, byte[]> attribute: attributes.entrySet()) {
885         attributeBuilder.setName(attribute.getKey());
886         attributeBuilder.setValue(ByteStringer.wrap(attribute.getValue()));
887         scanBuilder.addAttribute(attributeBuilder.build());
888       }
889     }
890     byte[] startRow = scan.getStartRow();
891     if (startRow != null && startRow.length > 0) {
892       scanBuilder.setStartRow(ByteStringer.wrap(startRow));
893     }
894     byte[] stopRow = scan.getStopRow();
895     if (stopRow != null && stopRow.length > 0) {
896       scanBuilder.setStopRow(ByteStringer.wrap(stopRow));
897     }
898     if (scan.hasFilter()) {
899       scanBuilder.setFilter(ProtobufUtil.toFilter(scan.getFilter()));
900     }
901     if (scan.hasFamilies()) {
902       Column.Builder columnBuilder = Column.newBuilder();
903       for (Map.Entry<byte[],NavigableSet<byte []>>
904           family: scan.getFamilyMap().entrySet()) {
905         columnBuilder.setFamily(ByteStringer.wrap(family.getKey()));
906         NavigableSet<byte []> qualifiers = family.getValue();
907         columnBuilder.clearQualifier();
908         if (qualifiers != null && qualifiers.size() > 0) {
909           for (byte [] qualifier: qualifiers) {
910             columnBuilder.addQualifier(ByteStringer.wrap(qualifier));
911           }
912         }
913         scanBuilder.addColumn(columnBuilder.build());
914       }
915     }
916     if (scan.getMaxResultsPerColumnFamily() >= 0) {
917       scanBuilder.setStoreLimit(scan.getMaxResultsPerColumnFamily());
918     }
919     if (scan.getRowOffsetPerColumnFamily() > 0) {
920       scanBuilder.setStoreOffset(scan.getRowOffsetPerColumnFamily());
921     }
922     if (scan.isReversed()) {
923       scanBuilder.setReversed(scan.isReversed());
924     }
925     if (scan.getConsistency() == Consistency.TIMELINE) {
926       scanBuilder.setConsistency(toConsistency(scan.getConsistency()));
927     }
928     if (scan.getCaching() > 0) {
929       scanBuilder.setCaching(scan.getCaching());
930     }
931     return scanBuilder.build();
932   }
933 
934   /**
935    * Convert a protocol buffer Scan to a client Scan
936    *
937    * @param proto the protocol buffer Scan to convert
938    * @return the converted client Scan
939    * @throws IOException
940    */
941   public static Scan toScan(
942       final ClientProtos.Scan proto) throws IOException {
943     byte [] startRow = HConstants.EMPTY_START_ROW;
944     byte [] stopRow  = HConstants.EMPTY_END_ROW;
945     if (proto.hasStartRow()) {
946       startRow = proto.getStartRow().toByteArray();
947     }
948     if (proto.hasStopRow()) {
949       stopRow = proto.getStopRow().toByteArray();
950     }
951     Scan scan = new Scan(startRow, stopRow);
952     if (proto.hasCacheBlocks()) {
953       scan.setCacheBlocks(proto.getCacheBlocks());
954     }
955     if (proto.hasMaxVersions()) {
956       scan.setMaxVersions(proto.getMaxVersions());
957     }
958     if (proto.hasStoreLimit()) {
959       scan.setMaxResultsPerColumnFamily(proto.getStoreLimit());
960     }
961     if (proto.hasStoreOffset()) {
962       scan.setRowOffsetPerColumnFamily(proto.getStoreOffset());
963     }
964     if (proto.hasLoadColumnFamiliesOnDemand()) {
965       scan.setLoadColumnFamiliesOnDemand(proto.getLoadColumnFamiliesOnDemand());
966     }
967     if (proto.hasTimeRange()) {
968       HBaseProtos.TimeRange timeRange = proto.getTimeRange();
969       long minStamp = 0;
970       long maxStamp = Long.MAX_VALUE;
971       if (timeRange.hasFrom()) {
972         minStamp = timeRange.getFrom();
973       }
974       if (timeRange.hasTo()) {
975         maxStamp = timeRange.getTo();
976       }
977       scan.setTimeRange(minStamp, maxStamp);
978     }
979     if (proto.hasFilter()) {
980       FilterProtos.Filter filter = proto.getFilter();
981       scan.setFilter(ProtobufUtil.toFilter(filter));
982     }
983     if (proto.hasBatchSize()) {
984       scan.setBatch(proto.getBatchSize());
985     }
986     if (proto.hasMaxResultSize()) {
987       scan.setMaxResultSize(proto.getMaxResultSize());
988     }
989     if (proto.hasSmall()) {
990       scan.setSmall(proto.getSmall());
991     }
992     for (NameBytesPair attribute: proto.getAttributeList()) {
993       scan.setAttribute(attribute.getName(), attribute.getValue().toByteArray());
994     }
995     if (proto.getColumnCount() > 0) {
996       for (Column column: proto.getColumnList()) {
997         byte[] family = column.getFamily().toByteArray();
998         if (column.getQualifierCount() > 0) {
999           for (ByteString qualifier: column.getQualifierList()) {
1000             scan.addColumn(family, qualifier.toByteArray());
1001           }
1002         } else {
1003           scan.addFamily(family);
1004         }
1005       }
1006     }
1007     if (proto.hasReversed()) {
1008       scan.setReversed(proto.getReversed());
1009     }
1010     if (proto.hasConsistency()) {
1011       scan.setConsistency(toConsistency(proto.getConsistency()));
1012     }
1013     if (proto.hasCaching()) {
1014       scan.setCaching(proto.getCaching());
1015     }
1016     return scan;
1017   }
1018 
1019   /**
1020    * Create a protocol buffer Get based on a client Get.
1021    *
1022    * @param get the client Get
1023    * @return a protocol buffer Get
1024    * @throws IOException
1025    */
1026   public static ClientProtos.Get toGet(
1027       final Get get) throws IOException {
1028     ClientProtos.Get.Builder builder =
1029       ClientProtos.Get.newBuilder();
1030     builder.setRow(ByteStringer.wrap(get.getRow()));
1031     builder.setCacheBlocks(get.getCacheBlocks());
1032     builder.setMaxVersions(get.getMaxVersions());
1033     if (get.getFilter() != null) {
1034       builder.setFilter(ProtobufUtil.toFilter(get.getFilter()));
1035     }
1036     TimeRange timeRange = get.getTimeRange();
1037     if (!timeRange.isAllTime()) {
1038       HBaseProtos.TimeRange.Builder timeRangeBuilder =
1039         HBaseProtos.TimeRange.newBuilder();
1040       timeRangeBuilder.setFrom(timeRange.getMin());
1041       timeRangeBuilder.setTo(timeRange.getMax());
1042       builder.setTimeRange(timeRangeBuilder.build());
1043     }
1044     Map<String, byte[]> attributes = get.getAttributesMap();
1045     if (!attributes.isEmpty()) {
1046       NameBytesPair.Builder attributeBuilder = NameBytesPair.newBuilder();
1047       for (Map.Entry<String, byte[]> attribute: attributes.entrySet()) {
1048         attributeBuilder.setName(attribute.getKey());
1049         attributeBuilder.setValue(ByteStringer.wrap(attribute.getValue()));
1050         builder.addAttribute(attributeBuilder.build());
1051       }
1052     }
1053     if (get.hasFamilies()) {
1054       Column.Builder columnBuilder = Column.newBuilder();
1055       Map<byte[], NavigableSet<byte[]>> families = get.getFamilyMap();
1056       for (Map.Entry<byte[], NavigableSet<byte[]>> family: families.entrySet()) {
1057         NavigableSet<byte[]> qualifiers = family.getValue();
1058         columnBuilder.setFamily(ByteStringer.wrap(family.getKey()));
1059         columnBuilder.clearQualifier();
1060         if (qualifiers != null && qualifiers.size() > 0) {
1061           for (byte[] qualifier: qualifiers) {
1062             columnBuilder.addQualifier(ByteStringer.wrap(qualifier));
1063           }
1064         }
1065         builder.addColumn(columnBuilder.build());
1066       }
1067     }
1068     if (get.getMaxResultsPerColumnFamily() >= 0) {
1069       builder.setStoreLimit(get.getMaxResultsPerColumnFamily());
1070     }
1071     if (get.getRowOffsetPerColumnFamily() > 0) {
1072       builder.setStoreOffset(get.getRowOffsetPerColumnFamily());
1073     }
1074     if (get.isCheckExistenceOnly()){
1075       builder.setExistenceOnly(true);
1076     }
1077     if (get.isClosestRowBefore()){
1078       builder.setClosestRowBefore(true);
1079     }
1080     if (get.getConsistency() != null && get.getConsistency() != Consistency.STRONG) {
1081       builder.setConsistency(toConsistency(get.getConsistency()));
1082     }
1083 
1084     return builder.build();
1085   }
1086 
1087   /**
1088    * Convert a client Increment to a protobuf Mutate.
1089    *
1090    * @param increment
1091    * @return the converted mutate
1092    */
1093   public static MutationProto toMutation(
1094     final Increment increment, final MutationProto.Builder builder, long nonce) {
1095     builder.setRow(ByteStringer.wrap(increment.getRow()));
1096     builder.setMutateType(MutationType.INCREMENT);
1097     builder.setDurability(toDurability(increment.getDurability()));
1098     if (nonce != HConstants.NO_NONCE) {
1099       builder.setNonce(nonce);
1100     }
1101     TimeRange timeRange = increment.getTimeRange();
1102     if (!timeRange.isAllTime()) {
1103       HBaseProtos.TimeRange.Builder timeRangeBuilder =
1104         HBaseProtos.TimeRange.newBuilder();
1105       timeRangeBuilder.setFrom(timeRange.getMin());
1106       timeRangeBuilder.setTo(timeRange.getMax());
1107       builder.setTimeRange(timeRangeBuilder.build());
1108     }
1109     ColumnValue.Builder columnBuilder = ColumnValue.newBuilder();
1110     QualifierValue.Builder valueBuilder = QualifierValue.newBuilder();
1111     for (Map.Entry<byte[], List<Cell>> family: increment.getFamilyCellMap().entrySet()) {
1112       columnBuilder.setFamily(ByteStringer.wrap(family.getKey()));
1113       columnBuilder.clearQualifierValue();
1114       List<Cell> values = family.getValue();
1115       if (values != null && values.size() > 0) {
1116         for (Cell cell: values) {
1117           valueBuilder.setQualifier(ByteStringer.wrap(
1118               cell.getQualifierArray(), cell.getQualifierOffset(), cell.getQualifierLength()));
1119           valueBuilder.setValue(ByteStringer.wrap(
1120               cell.getValueArray(), cell.getValueOffset(), cell.getValueLength()));
1121           if (cell.getTagsLength() > 0) {
1122             valueBuilder.setTags(ByteStringer.wrap(cell.getTagsArray(),
1123                 cell.getTagsOffset(), cell.getTagsLength()));
1124           }
1125           columnBuilder.addQualifierValue(valueBuilder.build());
1126         }
1127       }
1128       builder.addColumnValue(columnBuilder.build());
1129     }
1130     Map<String, byte[]> attributes = increment.getAttributesMap();
1131     if (!attributes.isEmpty()) {
1132       NameBytesPair.Builder attributeBuilder = NameBytesPair.newBuilder();
1133       for (Map.Entry<String, byte[]> attribute : attributes.entrySet()) {
1134         attributeBuilder.setName(attribute.getKey());
1135         attributeBuilder.setValue(ByteStringer.wrap(attribute.getValue()));
1136         builder.addAttribute(attributeBuilder.build());
1137       }
1138     }
1139     return builder.build();
1140   }
1141 
1142   public static MutationProto toMutation(final MutationType type, final Mutation mutation)
1143     throws IOException {
1144     return toMutation(type, mutation, HConstants.NO_NONCE);
1145   }
1146 
1147   /**
1148    * Create a protocol buffer Mutate based on a client Mutation
1149    *
1150    * @param type
1151    * @param mutation
1152    * @return a protobuf'd Mutation
1153    * @throws IOException
1154    */
1155   public static MutationProto toMutation(final MutationType type, final Mutation mutation,
1156     final long nonce) throws IOException {
1157     return toMutation(type, mutation, MutationProto.newBuilder(), nonce);
1158   }
1159 
1160   public static MutationProto toMutation(final MutationType type, final Mutation mutation,
1161       MutationProto.Builder builder) throws IOException {
1162     return toMutation(type, mutation, builder, HConstants.NO_NONCE);
1163   }
1164 
1165   @SuppressWarnings("deprecation")
1166   public static MutationProto toMutation(final MutationType type, final Mutation mutation,
1167       MutationProto.Builder builder, long nonce)
1168   throws IOException {
1169     builder = getMutationBuilderAndSetCommonFields(type, mutation, builder);
1170     if (nonce != HConstants.NO_NONCE) {
1171       builder.setNonce(nonce);
1172     }
1173     ColumnValue.Builder columnBuilder = ColumnValue.newBuilder();
1174     QualifierValue.Builder valueBuilder = QualifierValue.newBuilder();
1175     for (Map.Entry<byte[],List<Cell>> family: mutation.getFamilyCellMap().entrySet()) {
1176       columnBuilder.clear();
1177       columnBuilder.setFamily(ByteStringer.wrap(family.getKey()));
1178       for (Cell cell: family.getValue()) {
1179         valueBuilder.setQualifier(ByteStringer.wrap(
1180             cell.getQualifierArray(), cell.getQualifierOffset(), cell.getQualifierLength()));
1181         valueBuilder.setValue(ByteStringer.wrap(
1182             cell.getValueArray(), cell.getValueOffset(), cell.getValueLength()));
1183         valueBuilder.setTimestamp(cell.getTimestamp());
1184         if(cell.getTagsLength() > 0) {
1185           valueBuilder.setTags(ByteStringer.wrap(cell.getTagsArray(), cell.getTagsOffset(),
1186               cell.getTagsLength()));
1187         }
1188         if (type == MutationType.DELETE || (type == MutationType.PUT && CellUtil.isDelete(cell))) {
1189           KeyValue.Type keyValueType = KeyValue.Type.codeToType(cell.getTypeByte());
1190           valueBuilder.setDeleteType(toDeleteType(keyValueType));
1191         }
1192         columnBuilder.addQualifierValue(valueBuilder.build());
1193       }
1194       builder.addColumnValue(columnBuilder.build());
1195     }
1196     return builder.build();
1197   }
1198 
1199   /**
1200    * Create a protocol buffer MutationProto based on a client Mutation. Does NOT include data.
1201    * Understanding is that the Cell will be transported other than via protobuf.
1202    * @param type
1203    * @param mutation
1204    * @param builder
1205    * @return a protobuf'd Mutation
1206    * @throws IOException
1207    */
1208   public static MutationProto toMutationNoData(final MutationType type, final Mutation mutation,
1209       final MutationProto.Builder builder)  throws IOException {
1210     return toMutationNoData(type, mutation, builder, HConstants.NO_NONCE);
1211   }
1212 
1213   /**
1214    * Create a protocol buffer MutationProto based on a client Mutation.  Does NOT include data.
1215    * Understanding is that the Cell will be transported other than via protobuf.
1216    * @param type
1217    * @param mutation
1218    * @return a protobuf'd Mutation
1219    * @throws IOException
1220    */
1221   public static MutationProto toMutationNoData(final MutationType type, final Mutation mutation)
1222   throws IOException {
1223     MutationProto.Builder builder =  MutationProto.newBuilder();
1224     return toMutationNoData(type, mutation, builder);
1225   }
1226 
1227   public static MutationProto toMutationNoData(final MutationType type, final Mutation mutation,
1228       final MutationProto.Builder builder, long nonce) throws IOException {
1229     getMutationBuilderAndSetCommonFields(type, mutation, builder);
1230     builder.setAssociatedCellCount(mutation.size());
1231     if (nonce != HConstants.NO_NONCE) {
1232       builder.setNonce(nonce);
1233     }
1234     return builder.build();
1235   }
1236 
1237   /**
1238    * Code shared by {@link #toMutation(MutationType, Mutation)} and
1239    * {@link #toMutationNoData(MutationType, Mutation)}
1240    * @param type
1241    * @param mutation
1242    * @return A partly-filled out protobuf'd Mutation.
1243    */
1244   private static MutationProto.Builder getMutationBuilderAndSetCommonFields(final MutationType type,
1245       final Mutation mutation, MutationProto.Builder builder) {
1246     builder.setRow(ByteStringer.wrap(mutation.getRow()));
1247     builder.setMutateType(type);
1248     builder.setDurability(toDurability(mutation.getDurability()));
1249     builder.setTimestamp(mutation.getTimeStamp());
1250     Map<String, byte[]> attributes = mutation.getAttributesMap();
1251     if (!attributes.isEmpty()) {
1252       NameBytesPair.Builder attributeBuilder = NameBytesPair.newBuilder();
1253       for (Map.Entry<String, byte[]> attribute: attributes.entrySet()) {
1254         attributeBuilder.setName(attribute.getKey());
1255         attributeBuilder.setValue(ByteStringer.wrap(attribute.getValue()));
1256         builder.addAttribute(attributeBuilder.build());
1257       }
1258     }
1259     return builder;
1260   }
1261 
1262   /**
1263    * Convert a client Result to a protocol buffer Result
1264    *
1265    * @param result the client Result to convert
1266    * @return the converted protocol buffer Result
1267    */
1268   public static ClientProtos.Result toResult(final Result result) {
1269     if (result.getExists() != null) {
1270       return toResult(result.getExists(), result.isStale());
1271     }
1272 
1273     Cell[] cells = result.rawCells();
1274     if (cells == null || cells.length == 0) {
1275       return result.isStale() ? EMPTY_RESULT_PB_STALE : EMPTY_RESULT_PB;
1276     }
1277 
1278     ClientProtos.Result.Builder builder = ClientProtos.Result.newBuilder();
1279     for (Cell c : cells) {
1280       builder.addCell(toCell(c));
1281     }
1282 
1283     builder.setStale(result.isStale());
1284 
1285     return builder.build();
1286   }
1287 
1288   /**
1289    * Convert a client Result to a protocol buffer Result
1290    *
1291    * @param existence the client existence to send
1292    * @return the converted protocol buffer Result
1293    */
1294   public static ClientProtos.Result toResult(final boolean existence, boolean stale) {
1295     if (stale){
1296       return existence ? EMPTY_RESULT_PB_EXISTS_TRUE_STALE : EMPTY_RESULT_PB_EXISTS_FALSE_STALE;
1297     } else {
1298       return existence ? EMPTY_RESULT_PB_EXISTS_TRUE : EMPTY_RESULT_PB_EXISTS_FALSE;
1299     }
1300   }
1301 
1302   /**
1303    * Convert a client Result to a protocol buffer Result.
1304    * The pb Result does not include the Cell data.  That is for transport otherwise.
1305    *
1306    * @param result the client Result to convert
1307    * @return the converted protocol buffer Result
1308    */
1309   public static ClientProtos.Result toResultNoData(final Result result) {
1310     if (result.getExists() != null) return toResult(result.getExists(), result.isStale());
1311     int size = result.size();
1312     if (size == 0) return result.isStale() ? EMPTY_RESULT_PB_STALE : EMPTY_RESULT_PB;
1313     ClientProtos.Result.Builder builder = ClientProtos.Result.newBuilder();
1314     builder.setAssociatedCellCount(size);
1315     builder.setStale(result.isStale());
1316     return builder.build();
1317   }
1318 
1319   /**
1320    * Convert a protocol buffer Result to a client Result
1321    *
1322    * @param proto the protocol buffer Result to convert
1323    * @return the converted client Result
1324    */
1325   public static Result toResult(final ClientProtos.Result proto) {
1326     if (proto.hasExists()) {
1327       if (proto.getStale()) {
1328         return proto.getExists() ? EMPTY_RESULT_EXISTS_TRUE_STALE :EMPTY_RESULT_EXISTS_FALSE_STALE;
1329       }
1330       return proto.getExists() ? EMPTY_RESULT_EXISTS_TRUE : EMPTY_RESULT_EXISTS_FALSE;
1331     }
1332 
1333     List<CellProtos.Cell> values = proto.getCellList();
1334     if (values.isEmpty()){
1335       return proto.getStale() ? EMPTY_RESULT_STALE : EMPTY_RESULT;
1336     }
1337 
1338     List<Cell> cells = new ArrayList<Cell>(values.size());
1339     for (CellProtos.Cell c : values) {
1340       cells.add(toCell(c));
1341     }
1342     return Result.create(cells, null, proto.getStale());
1343   }
1344 
1345   /**
1346    * Convert a protocol buffer Result to a client Result
1347    *
1348    * @param proto the protocol buffer Result to convert
1349    * @param scanner Optional cell scanner.
1350    * @return the converted client Result
1351    * @throws IOException
1352    */
1353   public static Result toResult(final ClientProtos.Result proto, final CellScanner scanner)
1354   throws IOException {
1355     List<CellProtos.Cell> values = proto.getCellList();
1356 
1357     if (proto.hasExists()) {
1358       if ((values != null && !values.isEmpty()) ||
1359           (proto.hasAssociatedCellCount() && proto.getAssociatedCellCount() > 0)) {
1360         throw new IllegalArgumentException("bad proto: exists with cells is no allowed " + proto);
1361       }
1362       if (proto.getStale()) {
1363         return proto.getExists() ? EMPTY_RESULT_EXISTS_TRUE_STALE :EMPTY_RESULT_EXISTS_FALSE_STALE;
1364       }
1365       return proto.getExists() ? EMPTY_RESULT_EXISTS_TRUE : EMPTY_RESULT_EXISTS_FALSE;
1366     }
1367 
1368     // TODO: Unit test that has some Cells in scanner and some in the proto.
1369     List<Cell> cells = null;
1370     if (proto.hasAssociatedCellCount()) {
1371       int count = proto.getAssociatedCellCount();
1372       cells = new ArrayList<Cell>(count + values.size());
1373       for (int i = 0; i < count; i++) {
1374         if (!scanner.advance()) throw new IOException("Failed get " + i + " of " + count);
1375         cells.add(scanner.current());
1376       }
1377     }
1378 
1379     if (!values.isEmpty()){
1380       if (cells == null) cells = new ArrayList<Cell>(values.size());
1381       for (CellProtos.Cell c: values) {
1382         cells.add(toCell(c));
1383       }
1384     }
1385 
1386     return (cells == null || cells.isEmpty())
1387         ? (proto.getStale() ? EMPTY_RESULT_STALE : EMPTY_RESULT)
1388         : Result.create(cells, null, proto.getStale());
1389   }
1390 
1391 
1392   /**
1393    * Convert a ByteArrayComparable to a protocol buffer Comparator
1394    *
1395    * @param comparator the ByteArrayComparable to convert
1396    * @return the converted protocol buffer Comparator
1397    */
1398   public static ComparatorProtos.Comparator toComparator(ByteArrayComparable comparator) {
1399     ComparatorProtos.Comparator.Builder builder = ComparatorProtos.Comparator.newBuilder();
1400     builder.setName(comparator.getClass().getName());
1401     builder.setSerializedComparator(ByteStringer.wrap(comparator.toByteArray()));
1402     return builder.build();
1403   }
1404 
1405   /**
1406    * Convert a protocol buffer Comparator to a ByteArrayComparable
1407    *
1408    * @param proto the protocol buffer Comparator to convert
1409    * @return the converted ByteArrayComparable
1410    */
1411   @SuppressWarnings("unchecked")
1412   public static ByteArrayComparable toComparator(ComparatorProtos.Comparator proto)
1413   throws IOException {
1414     String type = proto.getName();
1415     String funcName = "parseFrom";
1416     byte [] value = proto.getSerializedComparator().toByteArray();
1417     try {
1418       Class<? extends ByteArrayComparable> c =
1419         (Class<? extends ByteArrayComparable>)Class.forName(type, true, CLASS_LOADER);
1420       Method parseFrom = c.getMethod(funcName, byte[].class);
1421       if (parseFrom == null) {
1422         throw new IOException("Unable to locate function: " + funcName + " in type: " + type);
1423       }
1424       return (ByteArrayComparable)parseFrom.invoke(null, value);
1425     } catch (Exception e) {
1426       throw new IOException(e);
1427     }
1428   }
1429 
1430   /**
1431    * Convert a protocol buffer Filter to a client Filter
1432    *
1433    * @param proto the protocol buffer Filter to convert
1434    * @return the converted Filter
1435    */
1436   @SuppressWarnings("unchecked")
1437   public static Filter toFilter(FilterProtos.Filter proto) throws IOException {
1438     String type = proto.getName();
1439     final byte [] value = proto.getSerializedFilter().toByteArray();
1440     String funcName = "parseFrom";
1441     try {
1442       Class<? extends Filter> c =
1443         (Class<? extends Filter>)Class.forName(type, true, CLASS_LOADER);
1444       Method parseFrom = c.getMethod(funcName, byte[].class);
1445       if (parseFrom == null) {
1446         throw new IOException("Unable to locate function: " + funcName + " in type: " + type);
1447       }
1448       return (Filter)parseFrom.invoke(c, value);
1449     } catch (Exception e) {
1450       // Either we couldn't instantiate the method object, or "parseFrom" failed.
1451       // In either case, let's not retry.
1452       throw new DoNotRetryIOException(e);
1453     }
1454   }
1455 
1456   /**
1457    * Convert a client Filter to a protocol buffer Filter
1458    *
1459    * @param filter the Filter to convert
1460    * @return the converted protocol buffer Filter
1461    */
1462   public static FilterProtos.Filter toFilter(Filter filter) throws IOException {
1463     FilterProtos.Filter.Builder builder = FilterProtos.Filter.newBuilder();
1464     builder.setName(filter.getClass().getName());
1465     builder.setSerializedFilter(ByteStringer.wrap(filter.toByteArray()));
1466     return builder.build();
1467   }
1468 
1469   /**
1470    * Convert a delete KeyValue type to protocol buffer DeleteType.
1471    *
1472    * @param type
1473    * @return protocol buffer DeleteType
1474    * @throws IOException
1475    */
1476   public static DeleteType toDeleteType(
1477       KeyValue.Type type) throws IOException {
1478     switch (type) {
1479     case Delete:
1480       return DeleteType.DELETE_ONE_VERSION;
1481     case DeleteColumn:
1482       return DeleteType.DELETE_MULTIPLE_VERSIONS;
1483     case DeleteFamily:
1484       return DeleteType.DELETE_FAMILY;
1485     case DeleteFamilyVersion:
1486       return DeleteType.DELETE_FAMILY_VERSION;
1487     default:
1488         throw new IOException("Unknown delete type: " + type);
1489     }
1490   }
1491 
1492   /**
1493    * Convert a protocol buffer DeleteType to delete KeyValue type.
1494    *
1495    * @param type The DeleteType
1496    * @return The type.
1497    * @throws IOException
1498    */
1499   public static KeyValue.Type fromDeleteType(
1500       DeleteType type) throws IOException {
1501     switch (type) {
1502     case DELETE_ONE_VERSION:
1503       return KeyValue.Type.Delete;
1504     case DELETE_MULTIPLE_VERSIONS:
1505       return KeyValue.Type.DeleteColumn;
1506     case DELETE_FAMILY:
1507       return KeyValue.Type.DeleteFamily;
1508     case DELETE_FAMILY_VERSION:
1509       return KeyValue.Type.DeleteFamilyVersion;
1510     default:
1511       throw new IOException("Unknown delete type: " + type);
1512     }
1513   }
1514 
1515   /**
1516    * Convert a stringified protocol buffer exception Parameter to a Java Exception
1517    *
1518    * @param parameter the protocol buffer Parameter to convert
1519    * @return the converted Exception
1520    * @throws IOException if failed to deserialize the parameter
1521    */
1522   @SuppressWarnings("unchecked")
1523   public static Throwable toException(final NameBytesPair parameter) throws IOException {
1524     if (parameter == null || !parameter.hasValue()) return null;
1525     String desc = parameter.getValue().toStringUtf8();
1526     String type = parameter.getName();
1527     try {
1528       Class<? extends Throwable> c =
1529         (Class<? extends Throwable>)Class.forName(type, true, CLASS_LOADER);
1530       Constructor<? extends Throwable> cn = null;
1531       try {
1532         cn = c.getDeclaredConstructor(String.class);
1533         return cn.newInstance(desc);
1534       } catch (NoSuchMethodException e) {
1535         // Could be a raw RemoteException. See HBASE-8987.
1536         cn = c.getDeclaredConstructor(String.class, String.class);
1537         return cn.newInstance(type, desc);
1538       }
1539     } catch (Exception e) {
1540       throw new IOException(e);
1541     }
1542   }
1543 
1544 // Start helpers for Client
1545 
1546   /**
1547    * A helper to get a row of the closet one before using client protocol.
1548    *
1549    * @param client
1550    * @param regionName
1551    * @param row
1552    * @param family
1553    * @return the row or the closestRowBefore if it doesn't exist
1554    * @throws IOException
1555    * @deprecated since 0.99 - use reversed scanner instead.
1556    */
1557   @Deprecated
1558   public static Result getRowOrBefore(final ClientService.BlockingInterface client,
1559       final byte[] regionName, final byte[] row,
1560       final byte[] family) throws IOException {
1561     GetRequest request =
1562       RequestConverter.buildGetRowOrBeforeRequest(
1563         regionName, row, family);
1564     try {
1565       GetResponse response = client.get(null, request);
1566       if (!response.hasResult()) return null;
1567       return toResult(response.getResult());
1568     } catch (ServiceException se) {
1569       throw getRemoteException(se);
1570     }
1571   }
1572 
1573   /**
1574    * A helper to bulk load a list of HFiles using client protocol.
1575    *
1576    * @param client
1577    * @param familyPaths
1578    * @param regionName
1579    * @param assignSeqNum
1580    * @return true if all are loaded
1581    * @throws IOException
1582    */
1583   public static boolean bulkLoadHFile(final ClientService.BlockingInterface client,
1584       final List<Pair<byte[], String>> familyPaths,
1585       final byte[] regionName, boolean assignSeqNum) throws IOException {
1586     BulkLoadHFileRequest request =
1587       RequestConverter.buildBulkLoadHFileRequest(familyPaths, regionName, assignSeqNum);
1588     try {
1589       BulkLoadHFileResponse response =
1590         client.bulkLoadHFile(null, request);
1591       return response.getLoaded();
1592     } catch (ServiceException se) {
1593       throw getRemoteException(se);
1594     }
1595   }
1596 
1597   public static CoprocessorServiceResponse execService(final ClientService.BlockingInterface client,
1598       final CoprocessorServiceCall call, final byte[] regionName) throws IOException {
1599     CoprocessorServiceRequest request = CoprocessorServiceRequest.newBuilder()
1600         .setCall(call).setRegion(
1601             RequestConverter.buildRegionSpecifier(REGION_NAME, regionName)).build();
1602     try {
1603       CoprocessorServiceResponse response =
1604           client.execService(null, request);
1605       return response;
1606     } catch (ServiceException se) {
1607       throw getRemoteException(se);
1608     }
1609   }
1610 
1611   public static CoprocessorServiceResponse execService(
1612     final MasterService.BlockingInterface client, final CoprocessorServiceCall call)
1613   throws IOException {
1614     CoprocessorServiceRequest request = CoprocessorServiceRequest.newBuilder()
1615         .setCall(call).setRegion(
1616             RequestConverter.buildRegionSpecifier(REGION_NAME, HConstants.EMPTY_BYTE_ARRAY)).build();
1617     try {
1618       CoprocessorServiceResponse response =
1619           client.execMasterService(null, request);
1620       return response;
1621     } catch (ServiceException se) {
1622       throw getRemoteException(se);
1623     }
1624   }
1625 
1626   /**
1627    * Make a region server endpoint call
1628    * @param client
1629    * @param call
1630    * @return CoprocessorServiceResponse
1631    * @throws IOException
1632    */
1633   public static CoprocessorServiceResponse execRegionServerService(
1634       final ClientService.BlockingInterface client, final CoprocessorServiceCall call)
1635       throws IOException {
1636     CoprocessorServiceRequest request =
1637         CoprocessorServiceRequest
1638             .newBuilder()
1639             .setCall(call)
1640             .setRegion(
1641               RequestConverter.buildRegionSpecifier(REGION_NAME, HConstants.EMPTY_BYTE_ARRAY))
1642             .build();
1643     try {
1644       CoprocessorServiceResponse response = client.execRegionServerService(null, request);
1645       return response;
1646     } catch (ServiceException se) {
1647       throw getRemoteException(se);
1648     }
1649   }
1650 
1651   @SuppressWarnings("unchecked")
1652   public static <T extends Service> T newServiceStub(Class<T> service, RpcChannel channel)
1653       throws Exception {
1654     return (T)Methods.call(service, null, "newStub",
1655         new Class[]{ RpcChannel.class }, new Object[]{ channel });
1656   }
1657 
1658 // End helpers for Client
1659 // Start helpers for Admin
1660 
1661   /**
1662    * A helper to retrieve region info given a region name
1663    * using admin protocol.
1664    *
1665    * @param admin
1666    * @param regionName
1667    * @return the retrieved region info
1668    * @throws IOException
1669    */
1670   public static HRegionInfo getRegionInfo(final AdminService.BlockingInterface admin,
1671       final byte[] regionName) throws IOException {
1672     try {
1673       GetRegionInfoRequest request =
1674         RequestConverter.buildGetRegionInfoRequest(regionName);
1675       GetRegionInfoResponse response =
1676         admin.getRegionInfo(null, request);
1677       return HRegionInfo.convert(response.getRegionInfo());
1678     } catch (ServiceException se) {
1679       throw getRemoteException(se);
1680     }
1681   }
1682 
1683   /**
1684    * A helper to close a region given a region name
1685    * using admin protocol.
1686    *
1687    * @param admin
1688    * @param regionName
1689    * @throws IOException
1690    */
1691   public static void closeRegion(final AdminService.BlockingInterface admin,
1692       final ServerName server, final byte[] regionName) throws IOException {
1693     CloseRegionRequest closeRegionRequest =
1694       RequestConverter.buildCloseRegionRequest(server, regionName);
1695     try {
1696       admin.closeRegion(null, closeRegionRequest);
1697     } catch (ServiceException se) {
1698       throw getRemoteException(se);
1699     }
1700   }
1701 
1702   /**
1703    * A helper to close a region given a region name
1704    * using admin protocol.
1705    *
1706    * @param admin
1707    * @param regionName
1708    * @return true if the region is closed
1709    * @throws IOException
1710    */
1711   public static boolean closeRegion(final AdminService.BlockingInterface admin,
1712       final ServerName server, final byte[] regionName,
1713       final ServerName destinationServer) throws IOException {
1714     CloseRegionRequest closeRegionRequest =
1715       RequestConverter.buildCloseRegionRequest(server,
1716         regionName, destinationServer);
1717     try {
1718       CloseRegionResponse response = admin.closeRegion(null, closeRegionRequest);
1719       return ResponseConverter.isClosed(response);
1720     } catch (ServiceException se) {
1721       throw getRemoteException(se);
1722     }
1723   }
1724 
1725 
1726   /**
1727    * A helper to open a region using admin protocol.
1728    * @param admin
1729    * @param region
1730    * @throws IOException
1731    */
1732   public static void openRegion(final AdminService.BlockingInterface admin,
1733       ServerName server, final HRegionInfo region) throws IOException {
1734     OpenRegionRequest request =
1735       RequestConverter.buildOpenRegionRequest(server, region, null, null);
1736     try {
1737       admin.openRegion(null, request);
1738     } catch (ServiceException se) {
1739       throw ProtobufUtil.getRemoteException(se);
1740     }
1741   }
1742 
1743   /**
1744    * A helper to get the all the online regions on a region
1745    * server using admin protocol.
1746    *
1747    * @param admin
1748    * @return a list of online region info
1749    * @throws IOException
1750    */
1751   public static List<HRegionInfo> getOnlineRegions(final AdminService.BlockingInterface admin)
1752   throws IOException {
1753     GetOnlineRegionRequest request = RequestConverter.buildGetOnlineRegionRequest();
1754     GetOnlineRegionResponse response = null;
1755     try {
1756       response = admin.getOnlineRegion(null, request);
1757     } catch (ServiceException se) {
1758       throw getRemoteException(se);
1759     }
1760     return getRegionInfos(response);
1761   }
1762 
1763   /**
1764    * Get the list of region info from a GetOnlineRegionResponse
1765    *
1766    * @param proto the GetOnlineRegionResponse
1767    * @return the list of region info or null if <code>proto</code> is null
1768    */
1769   static List<HRegionInfo> getRegionInfos(final GetOnlineRegionResponse proto) {
1770     if (proto == null) return null;
1771     List<HRegionInfo> regionInfos = new ArrayList<HRegionInfo>();
1772     for (RegionInfo regionInfo: proto.getRegionInfoList()) {
1773       regionInfos.add(HRegionInfo.convert(regionInfo));
1774     }
1775     return regionInfos;
1776   }
1777 
1778   /**
1779    * A helper to get the info of a region server using admin protocol.
1780    *
1781    * @param admin
1782    * @return the server name
1783    * @throws IOException
1784    */
1785   public static ServerInfo getServerInfo(final AdminService.BlockingInterface admin)
1786   throws IOException {
1787     GetServerInfoRequest request = RequestConverter.buildGetServerInfoRequest();
1788     try {
1789       GetServerInfoResponse response = admin.getServerInfo(null, request);
1790       return response.getServerInfo();
1791     } catch (ServiceException se) {
1792       throw getRemoteException(se);
1793     }
1794   }
1795 
1796   /**
1797    * A helper to get the list of files of a column family
1798    * on a given region using admin protocol.
1799    *
1800    * @param admin
1801    * @param regionName
1802    * @param family
1803    * @return the list of store files
1804    * @throws IOException
1805    */
1806   public static List<String> getStoreFiles(final AdminService.BlockingInterface admin,
1807       final byte[] regionName, final byte[] family)
1808   throws IOException {
1809     GetStoreFileRequest request =
1810       RequestConverter.buildGetStoreFileRequest(regionName, family);
1811     try {
1812       GetStoreFileResponse response = admin.getStoreFile(null, request);
1813       return response.getStoreFileList();
1814     } catch (ServiceException se) {
1815       throw ProtobufUtil.getRemoteException(se);
1816     }
1817   }
1818 
1819   /**
1820    * A helper to split a region using admin protocol.
1821    *
1822    * @param admin
1823    * @param hri
1824    * @param splitPoint
1825    * @throws IOException
1826    */
1827   public static void split(final AdminService.BlockingInterface admin,
1828       final HRegionInfo hri, byte[] splitPoint) throws IOException {
1829     SplitRegionRequest request =
1830       RequestConverter.buildSplitRegionRequest(hri.getRegionName(), splitPoint);
1831     try {
1832       admin.splitRegion(null, request);
1833     } catch (ServiceException se) {
1834       throw ProtobufUtil.getRemoteException(se);
1835     }
1836   }
1837 
1838   /**
1839    * A helper to merge regions using admin protocol. Send request to
1840    * regionserver.
1841    * @param admin
1842    * @param region_a
1843    * @param region_b
1844    * @param forcible true if do a compulsory merge, otherwise we will only merge
1845    *          two adjacent regions
1846    * @throws IOException
1847    */
1848   public static void mergeRegions(final AdminService.BlockingInterface admin,
1849       final HRegionInfo region_a, final HRegionInfo region_b,
1850       final boolean forcible) throws IOException {
1851     MergeRegionsRequest request = RequestConverter.buildMergeRegionsRequest(
1852         region_a.getRegionName(), region_b.getRegionName(),forcible);
1853     try {
1854       admin.mergeRegions(null, request);
1855     } catch (ServiceException se) {
1856       throw ProtobufUtil.getRemoteException(se);
1857     }
1858   }
1859 
1860 // End helpers for Admin
1861 
1862   /*
1863    * Get the total (read + write) requests from a RegionLoad pb
1864    * @param rl - RegionLoad pb
1865    * @return total (read + write) requests
1866    */
1867   public static long getTotalRequestsCount(RegionLoad rl) {
1868     if (rl == null) {
1869       return 0;
1870     }
1871 
1872     return rl.getReadRequestsCount() + rl.getWriteRequestsCount();
1873   }
1874 
1875 
1876   /**
1877    * @param m Message to get delimited pb serialization of (with pb magic prefix)
1878    */
1879   public static byte [] toDelimitedByteArray(final Message m) throws IOException {
1880     // Allocate arbitrary big size so we avoid resizing.
1881     ByteArrayOutputStream baos = new ByteArrayOutputStream(4096);
1882     baos.write(PB_MAGIC);
1883     m.writeDelimitedTo(baos);
1884     return baos.toByteArray();
1885   }
1886 
1887   /**
1888    * Converts a Permission proto to a client Permission object.
1889    *
1890    * @param proto the protobuf Permission
1891    * @return the converted Permission
1892    */
1893   public static Permission toPermission(AccessControlProtos.Permission proto) {
1894     if (proto.getType() != AccessControlProtos.Permission.Type.Global) {
1895       return toTablePermission(proto);
1896     } else {
1897       List<Permission.Action> actions = toPermissionActions(proto.getGlobalPermission().getActionList());
1898       return new Permission(actions.toArray(new Permission.Action[actions.size()]));
1899     }
1900   }
1901 
1902   /**
1903    * Converts a Permission proto to a client TablePermission object.
1904    *
1905    * @param proto the protobuf Permission
1906    * @return the converted TablePermission
1907    */
1908   public static TablePermission toTablePermission(AccessControlProtos.Permission proto) {
1909     if(proto.getType() == AccessControlProtos.Permission.Type.Global) {
1910       AccessControlProtos.GlobalPermission perm = proto.getGlobalPermission();
1911       List<Permission.Action> actions = toPermissionActions(perm.getActionList());
1912 
1913       return new TablePermission(null, null, null,
1914           actions.toArray(new Permission.Action[actions.size()]));
1915     }
1916     if(proto.getType() == AccessControlProtos.Permission.Type.Namespace) {
1917       AccessControlProtos.NamespacePermission perm = proto.getNamespacePermission();
1918       List<Permission.Action> actions = toPermissionActions(perm.getActionList());
1919 
1920       if(!proto.hasNamespacePermission()) {
1921         throw new IllegalStateException("Namespace must not be empty in NamespacePermission");
1922       }
1923       String namespace = perm.getNamespaceName().toStringUtf8();
1924       return new TablePermission(namespace, actions.toArray(new Permission.Action[actions.size()]));
1925     }
1926     if(proto.getType() == AccessControlProtos.Permission.Type.Table) {
1927       AccessControlProtos.TablePermission perm = proto.getTablePermission();
1928       List<Permission.Action> actions = toPermissionActions(perm.getActionList());
1929 
1930       byte[] qualifier = null;
1931       byte[] family = null;
1932       TableName table = null;
1933 
1934       if (!perm.hasTableName()) {
1935         throw new IllegalStateException("TableName cannot be empty");
1936       }
1937       table = ProtobufUtil.toTableName(perm.getTableName());
1938 
1939       if (perm.hasFamily()) family = perm.getFamily().toByteArray();
1940       if (perm.hasQualifier()) qualifier = perm.getQualifier().toByteArray();
1941 
1942       return new TablePermission(table, family, qualifier,
1943           actions.toArray(new Permission.Action[actions.size()]));
1944     }
1945     throw new IllegalStateException("Unrecognize Perm Type: "+proto.getType());
1946   }
1947 
1948   /**
1949    * Convert a client Permission to a Permission proto
1950    *
1951    * @param perm the client Permission
1952    * @return the protobuf Permission
1953    */
1954   public static AccessControlProtos.Permission toPermission(Permission perm) {
1955     AccessControlProtos.Permission.Builder ret = AccessControlProtos.Permission.newBuilder();
1956     if (perm instanceof TablePermission) {
1957       TablePermission tablePerm = (TablePermission)perm;
1958       if(tablePerm.hasNamespace()) {
1959         ret.setType(AccessControlProtos.Permission.Type.Namespace);
1960 
1961         AccessControlProtos.NamespacePermission.Builder builder =
1962             AccessControlProtos.NamespacePermission.newBuilder();
1963         builder.setNamespaceName(ByteString.copyFromUtf8(tablePerm.getNamespace()));
1964         Permission.Action actions[] = perm.getActions();
1965         if (actions != null) {
1966           for (Permission.Action a : actions) {
1967             builder.addAction(toPermissionAction(a));
1968           }
1969 	}
1970         ret.setNamespacePermission(builder);
1971         return ret.build();
1972       } else if (tablePerm.hasTable()) {
1973         ret.setType(AccessControlProtos.Permission.Type.Table);
1974 
1975         AccessControlProtos.TablePermission.Builder builder =
1976             AccessControlProtos.TablePermission.newBuilder();
1977         builder.setTableName(ProtobufUtil.toProtoTableName(tablePerm.getTableName()));
1978         if (tablePerm.hasFamily()) {
1979           builder.setFamily(ByteStringer.wrap(tablePerm.getFamily()));
1980         }
1981         if (tablePerm.hasQualifier()) {
1982           builder.setQualifier(ByteStringer.wrap(tablePerm.getQualifier()));
1983         }
1984         Permission.Action actions[] = perm.getActions();
1985         if (actions != null) {
1986           for (Permission.Action a : actions) {
1987             builder.addAction(toPermissionAction(a));
1988           }
1989         }
1990         ret.setTablePermission(builder);
1991         return ret.build();
1992       }
1993     }
1994 
1995     ret.setType(AccessControlProtos.Permission.Type.Global);
1996 
1997     AccessControlProtos.GlobalPermission.Builder builder =
1998         AccessControlProtos.GlobalPermission.newBuilder();
1999     Permission.Action actions[] = perm.getActions();
2000     if (actions != null) {
2001       for (Permission.Action a: actions) {
2002         builder.addAction(toPermissionAction(a));
2003       }
2004     }
2005     ret.setGlobalPermission(builder);
2006     return ret.build();
2007   }
2008 
2009   /**
2010    * Converts a list of Permission.Action proto to a list of client Permission.Action objects.
2011    *
2012    * @param protoActions the list of protobuf Actions
2013    * @return the converted list of Actions
2014    */
2015   public static List<Permission.Action> toPermissionActions(
2016       List<AccessControlProtos.Permission.Action> protoActions) {
2017     List<Permission.Action> actions = new ArrayList<Permission.Action>(protoActions.size());
2018     for (AccessControlProtos.Permission.Action a : protoActions) {
2019       actions.add(toPermissionAction(a));
2020     }
2021     return actions;
2022   }
2023 
2024   /**
2025    * Converts a Permission.Action proto to a client Permission.Action object.
2026    *
2027    * @param action the protobuf Action
2028    * @return the converted Action
2029    */
2030   public static Permission.Action toPermissionAction(
2031       AccessControlProtos.Permission.Action action) {
2032     switch (action) {
2033       case READ:
2034         return Permission.Action.READ;
2035       case WRITE:
2036         return Permission.Action.WRITE;
2037       case EXEC:
2038         return Permission.Action.EXEC;
2039       case CREATE:
2040         return Permission.Action.CREATE;
2041       case ADMIN:
2042         return Permission.Action.ADMIN;
2043     }
2044     throw new IllegalArgumentException("Unknown action value "+action.name());
2045   }
2046 
2047   /**
2048    * Convert a client Permission.Action to a Permission.Action proto
2049    *
2050    * @param action the client Action
2051    * @return the protobuf Action
2052    */
2053   public static AccessControlProtos.Permission.Action toPermissionAction(
2054       Permission.Action action) {
2055     switch (action) {
2056       case READ:
2057         return AccessControlProtos.Permission.Action.READ;
2058       case WRITE:
2059         return AccessControlProtos.Permission.Action.WRITE;
2060       case EXEC:
2061         return AccessControlProtos.Permission.Action.EXEC;
2062       case CREATE:
2063         return AccessControlProtos.Permission.Action.CREATE;
2064       case ADMIN:
2065         return AccessControlProtos.Permission.Action.ADMIN;
2066     }
2067     throw new IllegalArgumentException("Unknown action value "+action.name());
2068   }
2069 
2070   /**
2071    * Convert a client user permission to a user permission proto
2072    *
2073    * @param perm the client UserPermission
2074    * @return the protobuf UserPermission
2075    */
2076   public static AccessControlProtos.UserPermission toUserPermission(UserPermission perm) {
2077     return AccessControlProtos.UserPermission.newBuilder()
2078         .setUser(ByteStringer.wrap(perm.getUser()))
2079         .setPermission(toPermission(perm))
2080         .build();
2081   }
2082 
2083   /**
2084    * Converts a user permission proto to a client user permission object.
2085    *
2086    * @param proto the protobuf UserPermission
2087    * @return the converted UserPermission
2088    */
2089   public static UserPermission toUserPermission(AccessControlProtos.UserPermission proto) {
2090     return new UserPermission(proto.getUser().toByteArray(),
2091         toTablePermission(proto.getPermission()));
2092   }
2093 
2094   /**
2095    * Convert a ListMultimap<String, TablePermission> where key is username
2096    * to a protobuf UserPermission
2097    *
2098    * @param perm the list of user and table permissions
2099    * @return the protobuf UserTablePermissions
2100    */
2101   public static AccessControlProtos.UsersAndPermissions toUserTablePermissions(
2102       ListMultimap<String, TablePermission> perm) {
2103     AccessControlProtos.UsersAndPermissions.Builder builder =
2104                   AccessControlProtos.UsersAndPermissions.newBuilder();
2105     for (Map.Entry<String, Collection<TablePermission>> entry : perm.asMap().entrySet()) {
2106       AccessControlProtos.UsersAndPermissions.UserPermissions.Builder userPermBuilder =
2107                   AccessControlProtos.UsersAndPermissions.UserPermissions.newBuilder();
2108       userPermBuilder.setUser(ByteString.copyFromUtf8(entry.getKey()));
2109       for (TablePermission tablePerm: entry.getValue()) {
2110         userPermBuilder.addPermissions(toPermission(tablePerm));
2111       }
2112       builder.addUserPermissions(userPermBuilder.build());
2113     }
2114     return builder.build();
2115   }
2116 
2117   /**
2118    * A utility used to grant a user global permissions.
2119    * <p>
2120    * It's also called by the shell, in case you want to find references.
2121    *
2122    * @param protocol the AccessControlService protocol proxy
2123    * @param userShortName the short name of the user to grant permissions
2124    * @param actions the permissions to be granted
2125    * @throws ServiceException
2126    */
2127   public static void grant(AccessControlService.BlockingInterface protocol,
2128       String userShortName, Permission.Action... actions) throws ServiceException {
2129     List<AccessControlProtos.Permission.Action> permActions =
2130         Lists.newArrayListWithCapacity(actions.length);
2131     for (Permission.Action a : actions) {
2132       permActions.add(ProtobufUtil.toPermissionAction(a));
2133     }
2134     AccessControlProtos.GrantRequest request = RequestConverter.
2135       buildGrantRequest(userShortName, permActions.toArray(
2136         new AccessControlProtos.Permission.Action[actions.length]));
2137     protocol.grant(null, request);
2138   }
2139 
2140   /**
2141    * A utility used to grant a user table permissions. The permissions will
2142    * be for a table table/column family/qualifier.
2143    * <p>
2144    * It's also called by the shell, in case you want to find references.
2145    *
2146    * @param protocol the AccessControlService protocol proxy
2147    * @param userShortName the short name of the user to grant permissions
2148    * @param tableName optional table name
2149    * @param f optional column family
2150    * @param q optional qualifier
2151    * @param actions the permissions to be granted
2152    * @throws ServiceException
2153    */
2154   public static void grant(AccessControlService.BlockingInterface protocol,
2155       String userShortName, TableName tableName, byte[] f, byte[] q,
2156       Permission.Action... actions) throws ServiceException {
2157     List<AccessControlProtos.Permission.Action> permActions =
2158         Lists.newArrayListWithCapacity(actions.length);
2159     for (Permission.Action a : actions) {
2160       permActions.add(ProtobufUtil.toPermissionAction(a));
2161     }
2162     AccessControlProtos.GrantRequest request = RequestConverter.
2163       buildGrantRequest(userShortName, tableName, f, q, permActions.toArray(
2164         new AccessControlProtos.Permission.Action[actions.length]));
2165     protocol.grant(null, request);
2166   }
2167 
2168   /**
2169    * A utility used to grant a user namespace permissions.
2170    * <p>
2171    * It's also called by the shell, in case you want to find references.
2172    *
2173    * @param protocol the AccessControlService protocol proxy
2174    * @param namespace the short name of the user to grant permissions
2175    * @param actions the permissions to be granted
2176    * @throws ServiceException
2177    */
2178   public static void grant(AccessControlService.BlockingInterface protocol,
2179       String userShortName, String namespace,
2180       Permission.Action... actions) throws ServiceException {
2181     List<AccessControlProtos.Permission.Action> permActions =
2182         Lists.newArrayListWithCapacity(actions.length);
2183     for (Permission.Action a : actions) {
2184       permActions.add(ProtobufUtil.toPermissionAction(a));
2185     }
2186     AccessControlProtos.GrantRequest request = RequestConverter.
2187       buildGrantRequest(userShortName, namespace, permActions.toArray(
2188         new AccessControlProtos.Permission.Action[actions.length]));
2189     protocol.grant(null, request);
2190   }
2191 
2192   /**
2193    * A utility used to revoke a user's global permissions.
2194    * <p>
2195    * It's also called by the shell, in case you want to find references.
2196    *
2197    * @param protocol the AccessControlService protocol proxy
2198    * @param userShortName the short name of the user to revoke permissions
2199    * @param actions the permissions to be revoked
2200    * @throws ServiceException
2201    */
2202   public static void revoke(AccessControlService.BlockingInterface protocol,
2203       String userShortName, Permission.Action... actions) throws ServiceException {
2204     List<AccessControlProtos.Permission.Action> permActions =
2205         Lists.newArrayListWithCapacity(actions.length);
2206     for (Permission.Action a : actions) {
2207       permActions.add(ProtobufUtil.toPermissionAction(a));
2208     }
2209     AccessControlProtos.RevokeRequest request = RequestConverter.
2210       buildRevokeRequest(userShortName, permActions.toArray(
2211         new AccessControlProtos.Permission.Action[actions.length]));
2212     protocol.revoke(null, request);
2213   }
2214 
2215   /**
2216    * A utility used to revoke a user's table permissions. The permissions will
2217    * be for a table/column family/qualifier.
2218    * <p>
2219    * It's also called by the shell, in case you want to find references.
2220    *
2221    * @param protocol the AccessControlService protocol proxy
2222    * @param userShortName the short name of the user to revoke permissions
2223    * @param tableName optional table name
2224    * @param f optional column family
2225    * @param q optional qualifier
2226    * @param actions the permissions to be revoked
2227    * @throws ServiceException
2228    */
2229   public static void revoke(AccessControlService.BlockingInterface protocol,
2230       String userShortName, TableName tableName, byte[] f, byte[] q,
2231       Permission.Action... actions) throws ServiceException {
2232     List<AccessControlProtos.Permission.Action> permActions =
2233         Lists.newArrayListWithCapacity(actions.length);
2234     for (Permission.Action a : actions) {
2235       permActions.add(ProtobufUtil.toPermissionAction(a));
2236     }
2237     AccessControlProtos.RevokeRequest request = RequestConverter.
2238       buildRevokeRequest(userShortName, tableName, f, q, permActions.toArray(
2239         new AccessControlProtos.Permission.Action[actions.length]));
2240     protocol.revoke(null, request);
2241   }
2242 
2243   /**
2244    * A utility used to revoke a user's namespace permissions.
2245    * <p>
2246    * It's also called by the shell, in case you want to find references.
2247    *
2248    * @param protocol the AccessControlService protocol proxy
2249    * @param userShortName the short name of the user to revoke permissions
2250    * @param namespace optional table name
2251    * @param actions the permissions to be revoked
2252    * @throws ServiceException
2253    */
2254   public static void revoke(AccessControlService.BlockingInterface protocol,
2255       String userShortName, String namespace,
2256       Permission.Action... actions) throws ServiceException {
2257     List<AccessControlProtos.Permission.Action> permActions =
2258         Lists.newArrayListWithCapacity(actions.length);
2259     for (Permission.Action a : actions) {
2260       permActions.add(ProtobufUtil.toPermissionAction(a));
2261     }
2262     AccessControlProtos.RevokeRequest request = RequestConverter.
2263       buildRevokeRequest(userShortName, namespace, permActions.toArray(
2264         new AccessControlProtos.Permission.Action[actions.length]));
2265     protocol.revoke(null, request);
2266   }
2267 
2268   /**
2269    * A utility used to get user's global permissions.
2270    * <p>
2271    * It's also called by the shell, in case you want to find references.
2272    *
2273    * @param protocol the AccessControlService protocol proxy
2274    * @throws ServiceException
2275    */
2276   public static List<UserPermission> getUserPermissions(
2277       AccessControlService.BlockingInterface protocol) throws ServiceException {
2278     AccessControlProtos.GetUserPermissionsRequest.Builder builder =
2279       AccessControlProtos.GetUserPermissionsRequest.newBuilder();
2280     builder.setType(AccessControlProtos.Permission.Type.Global);
2281     AccessControlProtos.GetUserPermissionsRequest request = builder.build();
2282     AccessControlProtos.GetUserPermissionsResponse response =
2283       protocol.getUserPermissions(null, request);
2284     List<UserPermission> perms = new ArrayList<UserPermission>(response.getUserPermissionCount());
2285     for (AccessControlProtos.UserPermission perm: response.getUserPermissionList()) {
2286       perms.add(ProtobufUtil.toUserPermission(perm));
2287     }
2288     return perms;
2289   }
2290 
2291   /**
2292    * A utility used to get user table permissions.
2293    * <p>
2294    * It's also called by the shell, in case you want to find references.
2295    *
2296    * @param protocol the AccessControlService protocol proxy
2297    * @param t optional table name
2298    * @throws ServiceException
2299    */
2300   public static List<UserPermission> getUserPermissions(
2301       AccessControlService.BlockingInterface protocol,
2302       TableName t) throws ServiceException {
2303     AccessControlProtos.GetUserPermissionsRequest.Builder builder =
2304       AccessControlProtos.GetUserPermissionsRequest.newBuilder();
2305     if (t != null) {
2306       builder.setTableName(ProtobufUtil.toProtoTableName(t));
2307     }
2308     builder.setType(AccessControlProtos.Permission.Type.Table);
2309     AccessControlProtos.GetUserPermissionsRequest request = builder.build();
2310     AccessControlProtos.GetUserPermissionsResponse response =
2311       protocol.getUserPermissions(null, request);
2312     List<UserPermission> perms = new ArrayList<UserPermission>(response.getUserPermissionCount());
2313     for (AccessControlProtos.UserPermission perm: response.getUserPermissionList()) {
2314       perms.add(ProtobufUtil.toUserPermission(perm));
2315     }
2316     return perms;
2317   }
2318 
2319   /**
2320    * A utility used to get permissions for selected namespace.
2321    * <p>
2322    * It's also called by the shell, in case you want to find references.
2323    *
2324    * @param protocol the AccessControlService protocol proxy
2325    * @param namespace name of the namespace
2326    * @throws ServiceException
2327    */
2328   public static List<UserPermission> getUserPermissions(
2329       AccessControlService.BlockingInterface protocol,
2330       byte[] namespace) throws ServiceException {
2331     AccessControlProtos.GetUserPermissionsRequest.Builder builder =
2332       AccessControlProtos.GetUserPermissionsRequest.newBuilder();
2333     if (namespace != null) {
2334       builder.setNamespaceName(ByteStringer.wrap(namespace));
2335     }
2336     builder.setType(AccessControlProtos.Permission.Type.Namespace);
2337     AccessControlProtos.GetUserPermissionsRequest request = builder.build();
2338     AccessControlProtos.GetUserPermissionsResponse response =
2339       protocol.getUserPermissions(null, request);
2340     List<UserPermission> perms = new ArrayList<UserPermission>(response.getUserPermissionCount());
2341     for (AccessControlProtos.UserPermission perm: response.getUserPermissionList()) {
2342       perms.add(ProtobufUtil.toUserPermission(perm));
2343     }
2344     return perms;
2345   }
2346 
2347   /**
2348    * Convert a protobuf UserTablePermissions to a
2349    * ListMultimap<String, TablePermission> where key is username.
2350    *
2351    * @param proto the protobuf UserPermission
2352    * @return the converted UserPermission
2353    */
2354   public static ListMultimap<String, TablePermission> toUserTablePermissions(
2355       AccessControlProtos.UsersAndPermissions proto) {
2356     ListMultimap<String, TablePermission> perms = ArrayListMultimap.create();
2357     AccessControlProtos.UsersAndPermissions.UserPermissions userPerm;
2358 
2359     for (int i = 0; i < proto.getUserPermissionsCount(); i++) {
2360       userPerm = proto.getUserPermissions(i);
2361       for (int j = 0; j < userPerm.getPermissionsCount(); j++) {
2362         TablePermission tablePerm = toTablePermission(userPerm.getPermissions(j));
2363         perms.put(userPerm.getUser().toStringUtf8(), tablePerm);
2364       }
2365     }
2366 
2367     return perms;
2368   }
2369 
2370   /**
2371    * Converts a Token instance (with embedded identifier) to the protobuf representation.
2372    *
2373    * @param token the Token instance to copy
2374    * @return the protobuf Token message
2375    */
2376   public static AuthenticationProtos.Token toToken(Token<AuthenticationTokenIdentifier> token) {
2377     AuthenticationProtos.Token.Builder builder = AuthenticationProtos.Token.newBuilder();
2378     builder.setIdentifier(ByteStringer.wrap(token.getIdentifier()));
2379     builder.setPassword(ByteStringer.wrap(token.getPassword()));
2380     if (token.getService() != null) {
2381       builder.setService(ByteString.copyFromUtf8(token.getService().toString()));
2382     }
2383     return builder.build();
2384   }
2385 
2386   /**
2387    * Converts a protobuf Token message back into a Token instance.
2388    *
2389    * @param proto the protobuf Token message
2390    * @return the Token instance
2391    */
2392   public static Token<AuthenticationTokenIdentifier> toToken(AuthenticationProtos.Token proto) {
2393     return new Token<AuthenticationTokenIdentifier>(
2394         proto.hasIdentifier() ? proto.getIdentifier().toByteArray() : null,
2395         proto.hasPassword() ? proto.getPassword().toByteArray() : null,
2396         AuthenticationTokenIdentifier.AUTH_TOKEN_TYPE,
2397         proto.hasService() ? new Text(proto.getService().toStringUtf8()) : null);
2398   }
2399 
2400   /**
2401    * Find the HRegion encoded name based on a region specifier
2402    *
2403    * @param regionSpecifier the region specifier
2404    * @return the corresponding region's encoded name
2405    * @throws DoNotRetryIOException if the specifier type is unsupported
2406    */
2407   public static String getRegionEncodedName(
2408       final RegionSpecifier regionSpecifier) throws DoNotRetryIOException {
2409     byte[] value = regionSpecifier.getValue().toByteArray();
2410     RegionSpecifierType type = regionSpecifier.getType();
2411     switch (type) {
2412       case REGION_NAME:
2413         return HRegionInfo.encodeRegionName(value);
2414       case ENCODED_REGION_NAME:
2415         return Bytes.toString(value);
2416       default:
2417         throw new DoNotRetryIOException(
2418           "Unsupported region specifier type: " + type);
2419     }
2420   }
2421 
2422   public static ScanMetrics toScanMetrics(final byte[] bytes) {
2423     Parser<MapReduceProtos.ScanMetrics> parser = MapReduceProtos.ScanMetrics.PARSER;
2424     MapReduceProtos.ScanMetrics pScanMetrics = null;
2425     try {
2426       pScanMetrics = parser.parseFrom(bytes);
2427     } catch (InvalidProtocolBufferException e) {
2428       //Ignored there are just no key values to add.
2429     }
2430     ScanMetrics scanMetrics = new ScanMetrics();
2431     if (pScanMetrics != null) {
2432       for (HBaseProtos.NameInt64Pair pair : pScanMetrics.getMetricsList()) {
2433         if (pair.hasName() && pair.hasValue()) {
2434           scanMetrics.setCounter(pair.getName(), pair.getValue());
2435         }
2436       }
2437     }
2438     return scanMetrics;
2439   }
2440 
2441   public static MapReduceProtos.ScanMetrics toScanMetrics(ScanMetrics scanMetrics) {
2442     MapReduceProtos.ScanMetrics.Builder builder = MapReduceProtos.ScanMetrics.newBuilder();
2443     Map<String, Long> metrics = scanMetrics.getMetricsMap();
2444     for (Entry<String, Long> e : metrics.entrySet()) {
2445       HBaseProtos.NameInt64Pair nameInt64Pair =
2446           HBaseProtos.NameInt64Pair.newBuilder()
2447               .setName(e.getKey())
2448               .setValue(e.getValue())
2449               .build();
2450       builder.addMetrics(nameInt64Pair);
2451     }
2452     return builder.build();
2453   }
2454 
2455   /**
2456    * Unwraps an exception from a protobuf service into the underlying (expected) IOException.
2457    * This method will <strong>always</strong> throw an exception.
2458    * @param se the {@code ServiceException} instance to convert into an {@code IOException}
2459    */
2460   public static void toIOException(ServiceException se) throws IOException {
2461     if (se == null) {
2462       throw new NullPointerException("Null service exception passed!");
2463     }
2464 
2465     Throwable cause = se.getCause();
2466     if (cause != null && cause instanceof IOException) {
2467       throw (IOException)cause;
2468     }
2469     throw new IOException(se);
2470   }
2471 
2472   public static CellProtos.Cell toCell(final Cell kv) {
2473     // Doing this is going to kill us if we do it for all data passed.
2474     // St.Ack 20121205
2475     CellProtos.Cell.Builder kvbuilder = CellProtos.Cell.newBuilder();
2476     kvbuilder.setRow(ByteStringer.wrap(kv.getRowArray(), kv.getRowOffset(),
2477         kv.getRowLength()));
2478     kvbuilder.setFamily(ByteStringer.wrap(kv.getFamilyArray(),
2479         kv.getFamilyOffset(), kv.getFamilyLength()));
2480     kvbuilder.setQualifier(ByteStringer.wrap(kv.getQualifierArray(),
2481         kv.getQualifierOffset(), kv.getQualifierLength()));
2482     kvbuilder.setCellType(CellProtos.CellType.valueOf(kv.getTypeByte()));
2483     kvbuilder.setTimestamp(kv.getTimestamp());
2484     kvbuilder.setValue(ByteStringer.wrap(kv.getValueArray(), kv.getValueOffset(),
2485         kv.getValueLength()));
2486     return kvbuilder.build();
2487   }
2488 
2489   public static Cell toCell(final CellProtos.Cell cell) {
2490     // Doing this is going to kill us if we do it for all data passed.
2491     // St.Ack 20121205
2492     return CellUtil.createCell(cell.getRow().toByteArray(),
2493       cell.getFamily().toByteArray(),
2494       cell.getQualifier().toByteArray(),
2495       cell.getTimestamp(),
2496       (byte)cell.getCellType().getNumber(),
2497       cell.getValue().toByteArray());
2498   }
2499 
2500   public static HBaseProtos.NamespaceDescriptor toProtoNamespaceDescriptor(NamespaceDescriptor ns) {
2501     HBaseProtos.NamespaceDescriptor.Builder b =
2502         HBaseProtos.NamespaceDescriptor.newBuilder()
2503             .setName(ByteString.copyFromUtf8(ns.getName()));
2504     for(Map.Entry<String, String> entry: ns.getConfiguration().entrySet()) {
2505       b.addConfiguration(HBaseProtos.NameStringPair.newBuilder()
2506           .setName(entry.getKey())
2507           .setValue(entry.getValue()));
2508     }
2509     return b.build();
2510   }
2511 
2512   public static NamespaceDescriptor toNamespaceDescriptor(
2513       HBaseProtos.NamespaceDescriptor desc) throws IOException {
2514     NamespaceDescriptor.Builder b =
2515       NamespaceDescriptor.create(desc.getName().toStringUtf8());
2516     for(HBaseProtos.NameStringPair prop : desc.getConfigurationList()) {
2517       b.addConfiguration(prop.getName(), prop.getValue());
2518     }
2519     return b.build();
2520   }
2521 
2522   /**
2523    * Get an instance of the argument type declared in a class's signature. The
2524    * argument type is assumed to be a PB Message subclass, and the instance is
2525    * created using parseFrom method on the passed ByteString.
2526    * @param runtimeClass the runtime type of the class
2527    * @param position the position of the argument in the class declaration
2528    * @param b the ByteString which should be parsed to get the instance created
2529    * @return the instance
2530    * @throws IOException
2531    */
2532   @SuppressWarnings("unchecked")
2533   public static <T extends Message>
2534   T getParsedGenericInstance(Class<?> runtimeClass, int position, ByteString b)
2535       throws IOException {
2536     Type type = runtimeClass.getGenericSuperclass();
2537     Type argType = ((ParameterizedType)type).getActualTypeArguments()[position];
2538     Class<T> classType = (Class<T>)argType;
2539     T inst;
2540     try {
2541       Method m = classType.getMethod("parseFrom", ByteString.class);
2542       inst = (T)m.invoke(null, b);
2543       return inst;
2544     } catch (SecurityException e) {
2545       throw new IOException(e);
2546     } catch (NoSuchMethodException e) {
2547       throw new IOException(e);
2548     } catch (IllegalArgumentException e) {
2549       throw new IOException(e);
2550     } catch (InvocationTargetException e) {
2551       throw new IOException(e);
2552     } catch (IllegalAccessException e) {
2553       throw new IOException(e);
2554     }
2555   }
2556 
2557   @SuppressWarnings("deprecation")
2558   public static CompactionDescriptor toCompactionDescriptor(HRegionInfo info, byte[] family,
2559       List<Path> inputPaths, List<Path> outputPaths, Path storeDir) {
2560     // compaction descriptor contains relative paths.
2561     // input / output paths are relative to the store dir
2562     // store dir is relative to region dir
2563     CompactionDescriptor.Builder builder = CompactionDescriptor.newBuilder()
2564         .setTableName(ByteStringer.wrap(info.getTableName()))
2565         .setEncodedRegionName(ByteStringer.wrap(info.getEncodedNameAsBytes()))
2566         .setFamilyName(ByteStringer.wrap(family))
2567         .setStoreHomeDir(storeDir.getName()); //make relative
2568     for (Path inputPath : inputPaths) {
2569       builder.addCompactionInput(inputPath.getName()); //relative path
2570     }
2571     for (Path outputPath : outputPaths) {
2572       builder.addCompactionOutput(outputPath.getName());
2573     }
2574     builder.setRegionName(ByteStringer.wrap(info.getRegionName()));
2575     return builder.build();
2576   }
2577 
2578   public static FlushDescriptor toFlushDescriptor(FlushAction action, HRegionInfo hri,
2579       long flushSeqId, Map<byte[], List<Path>> committedFiles) {
2580     FlushDescriptor.Builder desc = FlushDescriptor.newBuilder()
2581         .setAction(action)
2582         .setEncodedRegionName(ByteStringer.wrap(hri.getEncodedNameAsBytes()))
2583         .setFlushSequenceNumber(flushSeqId)
2584         .setTableName(ByteStringer.wrap(hri.getTable().getName()));
2585 
2586     for (Map.Entry<byte[], List<Path>> entry : committedFiles.entrySet()) {
2587       WALProtos.FlushDescriptor.StoreFlushDescriptor.Builder builder =
2588           WALProtos.FlushDescriptor.StoreFlushDescriptor.newBuilder()
2589           .setFamilyName(ByteStringer.wrap(entry.getKey()))
2590           .setStoreHomeDir(Bytes.toString(entry.getKey())); //relative to region
2591       if (entry.getValue() != null) {
2592         for (Path path : entry.getValue()) {
2593           builder.addFlushOutput(path.getName());
2594         }
2595       }
2596       desc.addStoreFlushes(builder);
2597     }
2598     return desc.build();
2599   }
2600 
2601   public static RegionEventDescriptor toRegionEventDescriptor(
2602       EventType eventType, HRegionInfo hri, long seqId, ServerName server,
2603       Map<byte[], List<Path>> storeFiles) {
2604     RegionEventDescriptor.Builder desc = RegionEventDescriptor.newBuilder()
2605         .setEventType(eventType)
2606         .setTableName(ByteStringer.wrap(hri.getTable().getName()))
2607         .setEncodedRegionName(ByteStringer.wrap(hri.getEncodedNameAsBytes()))
2608         .setLogSequenceNumber(seqId)
2609         .setServer(toServerName(server));
2610 
2611     for (Map.Entry<byte[], List<Path>> entry : storeFiles.entrySet()) {
2612       RegionEventDescriptor.StoreDescriptor.Builder builder
2613         = RegionEventDescriptor.StoreDescriptor.newBuilder()
2614           .setFamilyName(ByteStringer.wrap(entry.getKey()))
2615           .setStoreHomeDir(Bytes.toString(entry.getKey()));
2616       for (Path path : entry.getValue()) {
2617         builder.addStoreFile(path.getName());
2618       }
2619 
2620       desc.addStores(builder);
2621     }
2622     return desc.build();
2623   }
2624 
2625   /**
2626    * Return short version of Message toString'd, shorter than TextFormat#shortDebugString.
2627    * Tries to NOT print out data both because it can be big but also so we do not have data in our
2628    * logs. Use judiciously.
2629    * @param m
2630    * @return toString of passed <code>m</code>
2631    */
2632   public static String getShortTextFormat(Message m) {
2633     if (m == null) return "null";
2634     if (m instanceof ScanRequest) {
2635       // This should be small and safe to output.  No data.
2636       return TextFormat.shortDebugString(m);
2637     } else if (m instanceof RegionServerReportRequest) {
2638       // Print a short message only, just the servername and the requests, not the full load.
2639       RegionServerReportRequest r = (RegionServerReportRequest)m;
2640       return "server " + TextFormat.shortDebugString(r.getServer()) +
2641         " load { numberOfRequests: " + r.getLoad().getNumberOfRequests() + " }";
2642     } else if (m instanceof RegionServerStartupRequest) {
2643       // Should be small enough.
2644       return TextFormat.shortDebugString(m);
2645     } else if (m instanceof MutationProto) {
2646       return toShortString((MutationProto)m);
2647     } else if (m instanceof GetRequest) {
2648       GetRequest r = (GetRequest) m;
2649       return "region= " + getStringForByteString(r.getRegion().getValue()) +
2650           ", row=" + getStringForByteString(r.getGet().getRow());
2651     } else if (m instanceof ClientProtos.MultiRequest) {
2652       ClientProtos.MultiRequest r = (ClientProtos.MultiRequest) m;
2653       // Get first set of Actions.
2654       ClientProtos.RegionAction actions = r.getRegionActionList().get(0);
2655       String row = actions.getActionCount() <= 0? "":
2656         getStringForByteString(actions.getAction(0).hasGet()?
2657           actions.getAction(0).getGet().getRow():
2658           actions.getAction(0).getMutation().getRow());
2659       return "region= " + getStringForByteString(actions.getRegion().getValue()) +
2660           ", for " + r.getRegionActionCount() +
2661           " actions and 1st row key=" + row;
2662     } else if (m instanceof ClientProtos.MutateRequest) {
2663       ClientProtos.MutateRequest r = (ClientProtos.MutateRequest) m;
2664       return "region= " + getStringForByteString(r.getRegion().getValue()) +
2665           ", row=" + getStringForByteString(r.getMutation().getRow());
2666     }
2667     return "TODO: " + m.getClass().toString();
2668   }
2669 
2670   private static String getStringForByteString(ByteString bs) {
2671     return Bytes.toStringBinary(bs.toByteArray());
2672   }
2673 
2674   /**
2675    * Print out some subset of a MutationProto rather than all of it and its data
2676    * @param proto Protobuf to print out
2677    * @return Short String of mutation proto
2678    */
2679   static String toShortString(final MutationProto proto) {
2680     return "row=" + Bytes.toString(proto.getRow().toByteArray()) +
2681         ", type=" + proto.getMutateType().toString();
2682   }
2683 
2684   public static TableName toTableName(HBaseProtos.TableName tableNamePB) {
2685     return TableName.valueOf(tableNamePB.getNamespace().asReadOnlyByteBuffer(),
2686         tableNamePB.getQualifier().asReadOnlyByteBuffer());
2687   }
2688 
2689   public static HBaseProtos.TableName toProtoTableName(TableName tableName) {
2690     return HBaseProtos.TableName.newBuilder()
2691         .setNamespace(ByteStringer.wrap(tableName.getNamespace()))
2692         .setQualifier(ByteStringer.wrap(tableName.getQualifier())).build();
2693   }
2694 
2695   public static TableName[] getTableNameArray(List<HBaseProtos.TableName> tableNamesList) {
2696     if (tableNamesList == null) {
2697       return new TableName[0];
2698     }
2699     TableName[] tableNames = new TableName[tableNamesList.size()];
2700     for (int i = 0; i < tableNamesList.size(); i++) {
2701       tableNames[i] = toTableName(tableNamesList.get(i));
2702     }
2703     return tableNames;
2704   }
2705 
2706   /**
2707    * Convert a protocol buffer CellVisibility to a client CellVisibility
2708    *
2709    * @param proto
2710    * @return the converted client CellVisibility
2711    */
2712   public static CellVisibility toCellVisibility(ClientProtos.CellVisibility proto) {
2713     if (proto == null) return null;
2714     return new CellVisibility(proto.getExpression());
2715   }
2716 
2717   /**
2718    * Convert a protocol buffer CellVisibility bytes to a client CellVisibility
2719    *
2720    * @param protoBytes
2721    * @return the converted client CellVisibility
2722    * @throws DeserializationException
2723    */
2724   public static CellVisibility toCellVisibility(byte[] protoBytes) throws DeserializationException {
2725     if (protoBytes == null) return null;
2726     ClientProtos.CellVisibility.Builder builder = ClientProtos.CellVisibility.newBuilder();
2727     ClientProtos.CellVisibility proto = null;
2728     try {
2729       proto = builder.mergeFrom(protoBytes).build();
2730     } catch (InvalidProtocolBufferException e) {
2731       throw new DeserializationException(e);
2732     }
2733     return toCellVisibility(proto);
2734   }
2735 
2736   /**
2737    * Create a protocol buffer CellVisibility based on a client CellVisibility.
2738    *
2739    * @param cellVisibility
2740    * @return a protocol buffer CellVisibility
2741    */
2742   public static ClientProtos.CellVisibility toCellVisibility(CellVisibility cellVisibility) {
2743     ClientProtos.CellVisibility.Builder builder = ClientProtos.CellVisibility.newBuilder();
2744     builder.setExpression(cellVisibility.getExpression());
2745     return builder.build();
2746   }
2747 
2748   /**
2749    * Convert a protocol buffer Authorizations to a client Authorizations
2750    *
2751    * @param proto
2752    * @return the converted client Authorizations
2753    */
2754   public static Authorizations toAuthorizations(ClientProtos.Authorizations proto) {
2755     if (proto == null) return null;
2756     return new Authorizations(proto.getLabelList());
2757   }
2758 
2759   /**
2760    * Convert a protocol buffer Authorizations bytes to a client Authorizations
2761    *
2762    * @param protoBytes
2763    * @return the converted client Authorizations
2764    * @throws DeserializationException
2765    */
2766   public static Authorizations toAuthorizations(byte[] protoBytes) throws DeserializationException {
2767     if (protoBytes == null) return null;
2768     ClientProtos.Authorizations.Builder builder = ClientProtos.Authorizations.newBuilder();
2769     ClientProtos.Authorizations proto = null;
2770     try {
2771       proto = builder.mergeFrom(protoBytes).build();
2772     } catch (InvalidProtocolBufferException e) {
2773       throw new DeserializationException(e);
2774     }
2775     return toAuthorizations(proto);
2776   }
2777 
2778   /**
2779    * Create a protocol buffer Authorizations based on a client Authorizations.
2780    *
2781    * @param authorizations
2782    * @return a protocol buffer Authorizations
2783    */
2784   public static ClientProtos.Authorizations toAuthorizations(Authorizations authorizations) {
2785     ClientProtos.Authorizations.Builder builder = ClientProtos.Authorizations.newBuilder();
2786     for (String label : authorizations.getLabels()) {
2787       builder.addLabel(label);
2788     }
2789     return builder.build();
2790   }
2791 
2792   public static AccessControlProtos.UsersAndPermissions toUsersAndPermissions(String user,
2793       Permission perms) {
2794     return AccessControlProtos.UsersAndPermissions.newBuilder()
2795       .addUserPermissions(AccessControlProtos.UsersAndPermissions.UserPermissions.newBuilder()
2796         .setUser(ByteString.copyFromUtf8(user))
2797         .addPermissions(toPermission(perms))
2798         .build())
2799       .build();
2800   }
2801 
2802   public static AccessControlProtos.UsersAndPermissions toUsersAndPermissions(
2803       ListMultimap<String, Permission> perms) {
2804     AccessControlProtos.UsersAndPermissions.Builder builder =
2805         AccessControlProtos.UsersAndPermissions.newBuilder();
2806     for (Map.Entry<String, Collection<Permission>> entry : perms.asMap().entrySet()) {
2807       AccessControlProtos.UsersAndPermissions.UserPermissions.Builder userPermBuilder =
2808         AccessControlProtos.UsersAndPermissions.UserPermissions.newBuilder();
2809       userPermBuilder.setUser(ByteString.copyFromUtf8(entry.getKey()));
2810       for (Permission perm: entry.getValue()) {
2811         userPermBuilder.addPermissions(toPermission(perm));
2812       }
2813       builder.addUserPermissions(userPermBuilder.build());
2814     }
2815     return builder.build();
2816   }
2817 
2818   public static ListMultimap<String, Permission> toUsersAndPermissions(
2819       AccessControlProtos.UsersAndPermissions proto) {
2820     ListMultimap<String, Permission> result = ArrayListMultimap.create();
2821     for (AccessControlProtos.UsersAndPermissions.UserPermissions userPerms:
2822         proto.getUserPermissionsList()) {
2823       String user = userPerms.getUser().toStringUtf8();
2824       for (AccessControlProtos.Permission perm: userPerms.getPermissionsList()) {
2825         result.put(user, toPermission(perm));
2826       }
2827     }
2828     return result;
2829   }
2830 
2831   /**
2832    * Convert a protocol buffer TimeUnit to a client TimeUnit
2833    *
2834    * @param proto
2835    * @return the converted client TimeUnit
2836    */
2837   public static TimeUnit toTimeUnit(final HBaseProtos.TimeUnit proto) {
2838     switch (proto) {
2839       case NANOSECONDS:  return TimeUnit.NANOSECONDS;
2840       case MICROSECONDS: return TimeUnit.MICROSECONDS;
2841       case MILLISECONDS: return TimeUnit.MILLISECONDS;
2842       case SECONDS:      return TimeUnit.SECONDS;
2843       case MINUTES:      return TimeUnit.MINUTES;
2844       case HOURS:        return TimeUnit.HOURS;
2845       case DAYS:         return TimeUnit.DAYS;
2846     }
2847     throw new RuntimeException("Invalid TimeUnit " + proto);
2848   }
2849 
2850   /**
2851    * Convert a client TimeUnit to a protocol buffer TimeUnit
2852    *
2853    * @param timeUnit
2854    * @return the converted protocol buffer TimeUnit
2855    */
2856   public static HBaseProtos.TimeUnit toProtoTimeUnit(final TimeUnit timeUnit) {
2857     switch (timeUnit) {
2858       case NANOSECONDS:  return HBaseProtos.TimeUnit.NANOSECONDS;
2859       case MICROSECONDS: return HBaseProtos.TimeUnit.MICROSECONDS;
2860       case MILLISECONDS: return HBaseProtos.TimeUnit.MILLISECONDS;
2861       case SECONDS:      return HBaseProtos.TimeUnit.SECONDS;
2862       case MINUTES:      return HBaseProtos.TimeUnit.MINUTES;
2863       case HOURS:        return HBaseProtos.TimeUnit.HOURS;
2864       case DAYS:         return HBaseProtos.TimeUnit.DAYS;
2865     }
2866     throw new RuntimeException("Invalid TimeUnit " + timeUnit);
2867   }
2868 
2869   /**
2870    * Convert a protocol buffer ThrottleType to a client ThrottleType
2871    *
2872    * @param proto
2873    * @return the converted client ThrottleType
2874    */
2875   public static ThrottleType toThrottleType(final QuotaProtos.ThrottleType proto) {
2876     switch (proto) {
2877       case REQUEST_NUMBER: return ThrottleType.REQUEST_NUMBER;
2878       case REQUEST_SIZE:   return ThrottleType.REQUEST_SIZE;
2879     }
2880     throw new RuntimeException("Invalid ThrottleType " + proto);
2881   }
2882 
2883   /**
2884    * Convert a client ThrottleType to a protocol buffer ThrottleType
2885    *
2886    * @param type
2887    * @return the converted protocol buffer ThrottleType
2888    */
2889   public static QuotaProtos.ThrottleType toProtoThrottleType(final ThrottleType type) {
2890     switch (type) {
2891       case REQUEST_NUMBER: return QuotaProtos.ThrottleType.REQUEST_NUMBER;
2892       case REQUEST_SIZE:   return QuotaProtos.ThrottleType.REQUEST_SIZE;
2893     }
2894     throw new RuntimeException("Invalid ThrottleType " + type);
2895   }
2896 
2897   /**
2898    * Convert a protocol buffer QuotaScope to a client QuotaScope
2899    *
2900    * @param proto
2901    * @return the converted client QuotaScope
2902    */
2903   public static QuotaScope toQuotaScope(final QuotaProtos.QuotaScope proto) {
2904     switch (proto) {
2905       case CLUSTER: return QuotaScope.CLUSTER;
2906       case MACHINE: return QuotaScope.MACHINE;
2907     }
2908     throw new RuntimeException("Invalid QuotaScope " + proto);
2909   }
2910 
2911   /**
2912    * Convert a client QuotaScope to a protocol buffer QuotaScope
2913    *
2914    * @param scope
2915    * @return the converted protocol buffer QuotaScope
2916    */
2917   public static QuotaProtos.QuotaScope toProtoQuotaScope(final QuotaScope scope) {
2918     switch (scope) {
2919       case CLUSTER: return QuotaProtos.QuotaScope.CLUSTER;
2920       case MACHINE: return QuotaProtos.QuotaScope.MACHINE;
2921     }
2922     throw new RuntimeException("Invalid QuotaScope " + scope);
2923   }
2924 
2925   /**
2926    * Convert a protocol buffer QuotaType to a client QuotaType
2927    *
2928    * @param proto
2929    * @return the converted client QuotaType
2930    */
2931   public static QuotaType toQuotaScope(final QuotaProtos.QuotaType proto) {
2932     switch (proto) {
2933       case THROTTLE: return QuotaType.THROTTLE;
2934     }
2935     throw new RuntimeException("Invalid QuotaType " + proto);
2936   }
2937 
2938   /**
2939    * Convert a client QuotaType to a protocol buffer QuotaType
2940    *
2941    * @param type
2942    * @return the converted protocol buffer QuotaType
2943    */
2944   public static QuotaProtos.QuotaType toProtoQuotaScope(final QuotaType type) {
2945     switch (type) {
2946       case THROTTLE: return QuotaProtos.QuotaType.THROTTLE;
2947     }
2948     throw new RuntimeException("Invalid QuotaType " + type);
2949   }
2950 
2951   /**
2952    * Build a protocol buffer TimedQuota
2953    *
2954    * @param limit the allowed number of request/data per timeUnit
2955    * @param timeUnit the limit time unit
2956    * @param scope the quota scope
2957    * @return the protocol buffer TimedQuota
2958    */
2959   public static QuotaProtos.TimedQuota toTimedQuota(final long limit, final TimeUnit timeUnit,
2960       final QuotaScope scope) {
2961     return QuotaProtos.TimedQuota.newBuilder()
2962             .setSoftLimit(limit)
2963             .setTimeUnit(toProtoTimeUnit(timeUnit))
2964             .setScope(toProtoQuotaScope(scope))
2965             .build();
2966   }
2967 }