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