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