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