2 * Licensed to the Apache Software Foundation (ASF) under one
3 * or more contributor license agreements. See the NOTICE file
4 * distributed with this work for additional information
5 * regarding copyright ownership. The ASF licenses this file
6 * to you under the Apache License, Version 2.0 (the
7 * "License"); you may not use this file except in compliance
8 * with the License. You may obtain a copy of the License at
10 * http://www.apache.org/licenses/LICENSE-2.0
12 * Unless required by applicable law or agreed to in writing, software
13 * distributed under the License is distributed on an "AS IS" BASIS,
14 * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
15 * See the License for the specific language governing permissions and
16 * limitations under the License.
18 package org
.apache
.hadoop
.hbase
.protobuf
;
20 import static org
.apache
.hadoop
.hbase
.protobuf
.ProtobufMagic
.PB_MAGIC
;
22 import com
.google
.protobuf
.ByteString
;
23 import com
.google
.protobuf
.CodedInputStream
;
24 import com
.google
.protobuf
.InvalidProtocolBufferException
;
25 import com
.google
.protobuf
.Message
;
26 import com
.google
.protobuf
.Parser
;
27 import com
.google
.protobuf
.RpcChannel
;
28 import com
.google
.protobuf
.RpcController
;
29 import com
.google
.protobuf
.Service
;
30 import com
.google
.protobuf
.ServiceException
;
31 import com
.google
.protobuf
.TextFormat
;
32 import java
.io
.IOException
;
33 import java
.lang
.reflect
.Constructor
;
34 import java
.lang
.reflect
.Method
;
35 import java
.security
.AccessController
;
36 import java
.security
.PrivilegedAction
;
37 import java
.util
.ArrayList
;
38 import java
.util
.List
;
40 import java
.util
.NavigableSet
;
41 import java
.util
.Objects
;
42 import java
.util
.function
.Function
;
43 import org
.apache
.hadoop
.conf
.Configuration
;
44 import org
.apache
.hadoop
.hbase
.Cell
;
45 import org
.apache
.hadoop
.hbase
.Cell
.Type
;
46 import org
.apache
.hadoop
.hbase
.CellBuilderType
;
47 import org
.apache
.hadoop
.hbase
.CellScanner
;
48 import org
.apache
.hadoop
.hbase
.CellUtil
;
49 import org
.apache
.hadoop
.hbase
.DoNotRetryIOException
;
50 import org
.apache
.hadoop
.hbase
.ExtendedCellBuilder
;
51 import org
.apache
.hadoop
.hbase
.ExtendedCellBuilderFactory
;
52 import org
.apache
.hadoop
.hbase
.HBaseConfiguration
;
53 import org
.apache
.hadoop
.hbase
.HBaseIOException
;
54 import org
.apache
.hadoop
.hbase
.HConstants
;
55 import org
.apache
.hadoop
.hbase
.KeyValue
;
56 import org
.apache
.hadoop
.hbase
.ServerName
;
57 import org
.apache
.hadoop
.hbase
.TableName
;
58 import org
.apache
.hadoop
.hbase
.client
.Append
;
59 import org
.apache
.hadoop
.hbase
.client
.Consistency
;
60 import org
.apache
.hadoop
.hbase
.client
.Delete
;
61 import org
.apache
.hadoop
.hbase
.client
.Durability
;
62 import org
.apache
.hadoop
.hbase
.client
.Get
;
63 import org
.apache
.hadoop
.hbase
.client
.Increment
;
64 import org
.apache
.hadoop
.hbase
.client
.Mutation
;
65 import org
.apache
.hadoop
.hbase
.client
.PackagePrivateFieldAccessor
;
66 import org
.apache
.hadoop
.hbase
.client
.Put
;
67 import org
.apache
.hadoop
.hbase
.client
.Result
;
68 import org
.apache
.hadoop
.hbase
.client
.Scan
;
69 import org
.apache
.hadoop
.hbase
.client
.SnapshotType
;
70 import org
.apache
.hadoop
.hbase
.client
.metrics
.ScanMetrics
;
71 import org
.apache
.hadoop
.hbase
.exceptions
.DeserializationException
;
72 import org
.apache
.hadoop
.hbase
.filter
.ByteArrayComparable
;
73 import org
.apache
.hadoop
.hbase
.filter
.Filter
;
74 import org
.apache
.hadoop
.hbase
.io
.TimeRange
;
75 import org
.apache
.hadoop
.hbase
.net
.Address
;
76 import org
.apache
.hadoop
.hbase
.protobuf
.generated
.AdminProtos
.AdminService
;
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
.ServerInfo
;
80 import org
.apache
.hadoop
.hbase
.protobuf
.generated
.CellProtos
;
81 import org
.apache
.hadoop
.hbase
.protobuf
.generated
.ClientProtos
;
82 import org
.apache
.hadoop
.hbase
.protobuf
.generated
.ClientProtos
.Column
;
83 import org
.apache
.hadoop
.hbase
.protobuf
.generated
.ClientProtos
.MutationProto
;
84 import org
.apache
.hadoop
.hbase
.protobuf
.generated
.ClientProtos
.MutationProto
.ColumnValue
;
85 import org
.apache
.hadoop
.hbase
.protobuf
.generated
.ClientProtos
.MutationProto
.ColumnValue
.QualifierValue
;
86 import org
.apache
.hadoop
.hbase
.protobuf
.generated
.ClientProtos
.MutationProto
.DeleteType
;
87 import org
.apache
.hadoop
.hbase
.protobuf
.generated
.ClientProtos
.MutationProto
.MutationType
;
88 import org
.apache
.hadoop
.hbase
.protobuf
.generated
.ComparatorProtos
;
89 import org
.apache
.hadoop
.hbase
.protobuf
.generated
.FilterProtos
;
90 import org
.apache
.hadoop
.hbase
.protobuf
.generated
.HBaseProtos
;
91 import org
.apache
.hadoop
.hbase
.protobuf
.generated
.HBaseProtos
.NameBytesPair
;
92 import org
.apache
.hadoop
.hbase
.protobuf
.generated
.HBaseProtos
.RegionSpecifier
;
93 import org
.apache
.hadoop
.hbase
.protobuf
.generated
.HBaseProtos
.RegionSpecifier
.RegionSpecifierType
;
94 import org
.apache
.hadoop
.hbase
.protobuf
.generated
.MapReduceProtos
;
95 import org
.apache
.hadoop
.hbase
.protobuf
.generated
.RSGroupProtos
;
96 import org
.apache
.hadoop
.hbase
.protobuf
.generated
.ZooKeeperProtos
;
97 import org
.apache
.hadoop
.hbase
.rsgroup
.RSGroupInfo
;
98 import org
.apache
.hadoop
.hbase
.util
.Addressing
;
99 import org
.apache
.hadoop
.hbase
.util
.ByteStringer
;
100 import org
.apache
.hadoop
.hbase
.util
.Bytes
;
101 import org
.apache
.hadoop
.hbase
.util
.DynamicClassLoader
;
102 import org
.apache
.hadoop
.hbase
.util
.ExceptionUtil
;
103 import org
.apache
.hadoop
.hbase
.util
.Methods
;
104 import org
.apache
.hadoop
.ipc
.RemoteException
;
105 import org
.apache
.yetus
.audience
.InterfaceAudience
;
109 * NOTE: This class OVERLAPS ProtobufUtil in the subpackage 'shaded'. The latter is used
110 * internally and has more methods. This Class is for Coprocessor Endpoints only though they
111 * should not be using this private class. It should not be depended upon. Most methods here
112 * are COPIED from the shaded ProtobufUtils with only difference being they refer to non-shaded
116 // TODO: Generate this class from the shaded version.
117 @InterfaceAudience.Private
// TODO: some clients (Hive, etc) use this class.
118 public final class ProtobufUtil
{
119 private ProtobufUtil() {
123 * Many results are simple: no cell, exists true or false. To save on object creations,
124 * we reuse them across calls.
126 // TODO: PICK THESE UP FROM THE SHADED PROTOBUF.
127 private final static Cell
[] EMPTY_CELL_ARRAY
= new Cell
[]{};
128 private final static Result EMPTY_RESULT
= Result
.create(EMPTY_CELL_ARRAY
);
129 final static Result EMPTY_RESULT_EXISTS_TRUE
= Result
.create(null, true);
130 final static Result EMPTY_RESULT_EXISTS_FALSE
= Result
.create(null, false);
131 private final static Result EMPTY_RESULT_STALE
= Result
.create(EMPTY_CELL_ARRAY
, null, true);
132 private final static Result EMPTY_RESULT_EXISTS_TRUE_STALE
133 = Result
.create((Cell
[])null, true, true);
134 private final static Result EMPTY_RESULT_EXISTS_FALSE_STALE
135 = Result
.create((Cell
[])null, false, true);
137 private final static ClientProtos
.Result EMPTY_RESULT_PB
;
138 private final static ClientProtos
.Result EMPTY_RESULT_PB_EXISTS_TRUE
;
139 private final static ClientProtos
.Result EMPTY_RESULT_PB_EXISTS_FALSE
;
140 private final static ClientProtos
.Result EMPTY_RESULT_PB_STALE
;
141 private final static ClientProtos
.Result EMPTY_RESULT_PB_EXISTS_TRUE_STALE
;
142 private final static ClientProtos
.Result EMPTY_RESULT_PB_EXISTS_FALSE_STALE
;
146 ClientProtos
.Result
.Builder builder
= ClientProtos
.Result
.newBuilder();
148 builder
.setExists(true);
149 builder
.setAssociatedCellCount(0);
150 EMPTY_RESULT_PB_EXISTS_TRUE
= builder
.build();
152 builder
.setStale(true);
153 EMPTY_RESULT_PB_EXISTS_TRUE_STALE
= builder
.build();
156 builder
.setExists(false);
157 builder
.setAssociatedCellCount(0);
158 EMPTY_RESULT_PB_EXISTS_FALSE
= builder
.build();
159 builder
.setStale(true);
160 EMPTY_RESULT_PB_EXISTS_FALSE_STALE
= builder
.build();
163 builder
.setAssociatedCellCount(0);
164 EMPTY_RESULT_PB
= builder
.build();
165 builder
.setStale(true);
166 EMPTY_RESULT_PB_STALE
= builder
.build();
170 * Dynamic class loader to load filter/comparators
172 private final static class ClassLoaderHolder
{
173 private final static ClassLoader CLASS_LOADER
;
176 ClassLoader parent
= ProtobufUtil
.class.getClassLoader();
177 Configuration conf
= HBaseConfiguration
.create();
178 CLASS_LOADER
= AccessController
.doPrivileged((PrivilegedAction
<ClassLoader
>)
179 () -> new DynamicClassLoader(conf
, parent
)
185 * Prepend the passed bytes with four bytes of magic, {@link ProtobufMagic#PB_MAGIC},
186 * to flag what follows as a protobuf in hbase. Prepend these bytes to all content written to
188 * @param bytes Bytes to decorate
189 * @return The passed <code>bytes</code> with magic prepended (Creates a new
190 * byte array that is <code>bytes.length</code> plus {@link ProtobufMagic#PB_MAGIC}.length.
192 public static byte [] prependPBMagic(final byte [] bytes
) {
193 return Bytes
.add(PB_MAGIC
, bytes
);
197 * @param bytes Bytes to check.
198 * @return True if passed <code>bytes</code> has {@link ProtobufMagic#PB_MAGIC} for a prefix.
200 public static boolean isPBMagicPrefix(final byte [] bytes
) {
201 return ProtobufMagic
.isPBMagicPrefix(bytes
);
205 * @param bytes Bytes to check.
206 * @param offset offset to start at
207 * @param len length to use
208 * @return True if passed <code>bytes</code> has {@link ProtobufMagic#PB_MAGIC} for a prefix.
210 public static boolean isPBMagicPrefix(final byte [] bytes
, int offset
, int len
) {
211 return ProtobufMagic
.isPBMagicPrefix(bytes
, offset
, len
);
215 * @param bytes bytes to check
216 * @throws DeserializationException if we are missing the pb magic prefix
218 public static void expectPBMagicPrefix(final byte[] bytes
) throws DeserializationException
{
219 if (!isPBMagicPrefix(bytes
)) {
220 String bytesPrefix
= bytes
== null ?
"null" : Bytes
.toStringBinary(bytes
, 0, PB_MAGIC
.length
);
221 throw new DeserializationException(
222 "Missing pb magic " + Bytes
.toString(PB_MAGIC
) + " prefix, bytes: " + bytesPrefix
);
227 * @return Length of {@link ProtobufMagic#lengthOfPBMagic()}
229 public static int lengthOfPBMagic() {
230 return ProtobufMagic
.lengthOfPBMagic();
234 * Return the IOException thrown by the remote server wrapped in
235 * ServiceException as cause.
237 * @param se ServiceException that wraps IO exception thrown by the server
238 * @return Exception wrapped in ServiceException or
239 * a new IOException that wraps the unexpected ServiceException.
241 public static IOException
getRemoteException(ServiceException se
) {
242 return makeIOExceptionOfException(se
);
246 * Return the Exception thrown by the remote server wrapped in
247 * ServiceException as cause. RemoteException are left untouched.
249 * @param e ServiceException that wraps IO exception thrown by the server
250 * @return Exception wrapped in ServiceException.
252 public static IOException
getServiceException(org
.apache
.hbase
.thirdparty
.com
.google
.protobuf
.ServiceException e
) {
253 Throwable t
= e
.getCause();
254 if (ExceptionUtil
.isInterrupt(t
)) {
255 return ExceptionUtil
.asInterrupt(t
);
257 return t
instanceof IOException ?
(IOException
) t
: new HBaseIOException(t
);
261 * Like {@link #getRemoteException(ServiceException)} but more generic, able to handle more than
262 * just {@link ServiceException}. Prefer this method to
263 * {@link #getRemoteException(ServiceException)} because trying to
264 * contain direct protobuf references.
266 public static IOException
handleRemoteException(Throwable e
) {
267 return makeIOExceptionOfException(e
);
270 private static IOException
makeIOExceptionOfException(Throwable e
) {
272 if (e
instanceof ServiceException
||
273 e
instanceof org
.apache
.hbase
.thirdparty
.com
.google
.protobuf
.ServiceException
) {
276 if (ExceptionUtil
.isInterrupt(t
)) {
277 return ExceptionUtil
.asInterrupt(t
);
279 if (t
instanceof RemoteException
) {
280 t
= ((RemoteException
)t
).unwrapRemoteException();
282 return t
instanceof IOException?
(IOException
)t
: new HBaseIOException(t
);
286 * Convert a ServerName to a protocol buffer ServerName
288 * @param serverName the ServerName to convert
289 * @return the converted protocol buffer ServerName
290 * @see #toServerName(org.apache.hadoop.hbase.protobuf.generated.HBaseProtos.ServerName)
292 public static HBaseProtos
.ServerName
293 toServerName(final ServerName serverName
) {
294 if (serverName
== null) return null;
295 HBaseProtos
.ServerName
.Builder builder
=
296 HBaseProtos
.ServerName
.newBuilder();
297 builder
.setHostName(serverName
.getHostname());
298 if (serverName
.getPort() >= 0) {
299 builder
.setPort(serverName
.getPort());
301 if (serverName
.getStartcode() >= 0) {
302 builder
.setStartCode(serverName
.getStartcode());
304 return builder
.build();
308 * Convert a protocol buffer ServerName to a ServerName
310 * @param proto the protocol buffer ServerName to convert
311 * @return the converted ServerName
313 public static ServerName
toServerName(final HBaseProtos
.ServerName proto
) {
314 if (proto
== null) return null;
315 String hostName
= proto
.getHostName();
318 if (proto
.hasPort()) {
319 port
= proto
.getPort();
321 if (proto
.hasStartCode()) {
322 startCode
= proto
.getStartCode();
324 return ServerName
.valueOf(hostName
, port
, startCode
);
328 * Convert a protobuf Durability into a client Durability
330 public static Durability
toDurability(
331 final ClientProtos
.MutationProto
.Durability proto
) {
334 return Durability
.USE_DEFAULT
;
336 return Durability
.SKIP_WAL
;
338 return Durability
.ASYNC_WAL
;
340 return Durability
.SYNC_WAL
;
342 return Durability
.FSYNC_WAL
;
344 return Durability
.USE_DEFAULT
;
349 * Convert a client Durability into a protbuf Durability
351 public static ClientProtos
.MutationProto
.Durability
toDurability(
352 final Durability d
) {
355 return ClientProtos
.MutationProto
.Durability
.USE_DEFAULT
;
357 return ClientProtos
.MutationProto
.Durability
.SKIP_WAL
;
359 return ClientProtos
.MutationProto
.Durability
.ASYNC_WAL
;
361 return ClientProtos
.MutationProto
.Durability
.SYNC_WAL
;
363 return ClientProtos
.MutationProto
.Durability
.FSYNC_WAL
;
365 return ClientProtos
.MutationProto
.Durability
.USE_DEFAULT
;
370 * Convert a protocol buffer Get to a client Get
372 * @param proto the protocol buffer Get to convert
373 * @return the converted client Get
374 * @throws IOException
376 public static Get
toGet(final ClientProtos
.Get proto
) throws IOException
{
377 if (proto
== null) return null;
378 byte[] row
= proto
.getRow().toByteArray();
379 Get get
= new Get(row
);
380 if (proto
.hasCacheBlocks()) {
381 get
.setCacheBlocks(proto
.getCacheBlocks());
383 if (proto
.hasMaxVersions()) {
384 get
.readVersions(proto
.getMaxVersions());
386 if (proto
.hasStoreLimit()) {
387 get
.setMaxResultsPerColumnFamily(proto
.getStoreLimit());
389 if (proto
.hasStoreOffset()) {
390 get
.setRowOffsetPerColumnFamily(proto
.getStoreOffset());
392 if (proto
.getCfTimeRangeCount() > 0) {
393 for (HBaseProtos
.ColumnFamilyTimeRange cftr
: proto
.getCfTimeRangeList()) {
394 TimeRange timeRange
= protoToTimeRange(cftr
.getTimeRange());
395 get
.setColumnFamilyTimeRange(cftr
.getColumnFamily().toByteArray(),
396 timeRange
.getMin(), timeRange
.getMax());
399 if (proto
.hasTimeRange()) {
400 TimeRange timeRange
= protoToTimeRange(proto
.getTimeRange());
401 get
.setTimeRange(timeRange
.getMin(), timeRange
.getMax());
403 if (proto
.hasFilter()) {
404 FilterProtos
.Filter filter
= proto
.getFilter();
405 get
.setFilter(ProtobufUtil
.toFilter(filter
));
407 for (NameBytesPair attribute
: proto
.getAttributeList()) {
408 get
.setAttribute(attribute
.getName(), attribute
.getValue().toByteArray());
410 if (proto
.getColumnCount() > 0) {
411 for (Column column
: proto
.getColumnList()) {
412 byte[] family
= column
.getFamily().toByteArray();
413 if (column
.getQualifierCount() > 0) {
414 for (ByteString qualifier
: column
.getQualifierList()) {
415 get
.addColumn(family
, qualifier
.toByteArray());
418 get
.addFamily(family
);
422 if (proto
.hasExistenceOnly() && proto
.getExistenceOnly()){
423 get
.setCheckExistenceOnly(true);
425 if (proto
.hasConsistency()) {
426 get
.setConsistency(toConsistency(proto
.getConsistency()));
428 if (proto
.hasLoadColumnFamiliesOnDemand()) {
429 get
.setLoadColumnFamiliesOnDemand(proto
.getLoadColumnFamiliesOnDemand());
434 public static Consistency
toConsistency(ClientProtos
.Consistency consistency
) {
435 switch (consistency
) {
436 case STRONG
: return Consistency
.STRONG
;
437 case TIMELINE
: return Consistency
.TIMELINE
;
438 default : return Consistency
.STRONG
;
442 public static ClientProtos
.Consistency
toConsistency(Consistency consistency
) {
443 switch (consistency
) {
444 case STRONG
: return ClientProtos
.Consistency
.STRONG
;
445 case TIMELINE
: return ClientProtos
.Consistency
.TIMELINE
;
446 default : return ClientProtos
.Consistency
.STRONG
;
451 * Convert a protocol buffer Mutate to a Put.
453 * @param proto The protocol buffer MutationProto to convert
454 * @return A client Put.
455 * @throws IOException
457 public static Put
toPut(final MutationProto proto
)
459 return toPut(proto
, null);
463 * Convert a protocol buffer Mutate to a Put.
465 * @param proto The protocol buffer MutationProto to convert
466 * @param cellScanner If non-null, the Cell data that goes with this proto.
467 * @return A client Put.
468 * @throws IOException
470 public static Put
toPut(final MutationProto proto
, final CellScanner cellScanner
)
472 // TODO: Server-side at least why do we convert back to the Client types? Why not just pb it?
473 MutationType type
= proto
.getMutateType();
474 assert type
== MutationType
.PUT
: type
.name();
475 long timestamp
= proto
.hasTimestamp()? proto
.getTimestamp(): HConstants
.LATEST_TIMESTAMP
;
476 Put put
= proto
.hasRow() ?
new Put(proto
.getRow().toByteArray(), timestamp
) : null;
477 int cellCount
= proto
.hasAssociatedCellCount()? proto
.getAssociatedCellCount(): 0;
479 // The proto has metadata only and the data is separate to be found in the cellScanner.
480 if (cellScanner
== null) {
481 throw new DoNotRetryIOException("Cell count of " + cellCount
+ " but no cellScanner: " +
482 toShortString(proto
));
484 for (int i
= 0; i
< cellCount
; i
++) {
485 if (!cellScanner
.advance()) {
486 throw new DoNotRetryIOException("Cell count of " + cellCount
+ " but at index " + i
+
487 " no cell returned: " + toShortString(proto
));
489 Cell cell
= cellScanner
.current();
491 put
= new Put(cell
.getRowArray(), cell
.getRowOffset(), cell
.getRowLength(), timestamp
);
497 throw new IllegalArgumentException("row cannot be null");
499 // The proto has the metadata and the data itself
500 ExtendedCellBuilder cellBuilder
= ExtendedCellBuilderFactory
.create(CellBuilderType
.SHALLOW_COPY
);
501 for (ColumnValue column
: proto
.getColumnValueList()) {
502 byte[] family
= column
.getFamily().toByteArray();
503 for (QualifierValue qv
: column
.getQualifierValueList()) {
504 if (!qv
.hasValue()) {
505 throw new DoNotRetryIOException(
506 "Missing required field: qualifier value");
509 if (qv
.hasTimestamp()) {
510 ts
= qv
.getTimestamp();
514 allTagsBytes
= qv
.getTags().toByteArray();
515 if(qv
.hasDeleteType()) {
516 put
.add(cellBuilder
.clear()
517 .setRow(put
.getRow())
519 .setQualifier(qv
.hasQualifier() ? qv
.getQualifier().toByteArray() : null)
521 .setType(fromDeleteType(qv
.getDeleteType()).getCode())
522 .setTags(allTagsBytes
)
525 put
.add(cellBuilder
.clear()
526 .setRow(put
.getRow())
528 .setQualifier(qv
.hasQualifier() ? qv
.getQualifier().toByteArray() : null)
530 .setType(Cell
.Type
.Put
)
531 .setValue(qv
.hasValue() ? qv
.getValue().toByteArray() : null)
532 .setTags(allTagsBytes
)
536 if(qv
.hasDeleteType()) {
537 put
.add(cellBuilder
.clear()
538 .setRow(put
.getRow())
540 .setQualifier(qv
.hasQualifier() ? qv
.getQualifier().toByteArray() : null)
542 .setType(fromDeleteType(qv
.getDeleteType()).getCode())
545 put
.add(cellBuilder
.clear()
546 .setRow(put
.getRow())
548 .setQualifier(qv
.hasQualifier() ? qv
.getQualifier().toByteArray() : null)
551 .setValue(qv
.hasValue() ? qv
.getValue().toByteArray() : null)
558 put
.setDurability(toDurability(proto
.getDurability()));
559 for (NameBytesPair attribute
: proto
.getAttributeList()) {
560 put
.setAttribute(attribute
.getName(), attribute
.getValue().toByteArray());
566 * Convert a protocol buffer Mutate to a Delete
568 * @param proto the protocol buffer Mutate to convert
569 * @return the converted client Delete
570 * @throws IOException
572 public static Delete
toDelete(final MutationProto proto
)
574 return toDelete(proto
, null);
578 * Convert a protocol buffer Mutate to a Delete
580 * @param proto the protocol buffer Mutate to convert
581 * @param cellScanner if non-null, the data that goes with this delete.
582 * @return the converted client Delete
583 * @throws IOException
585 public static Delete
toDelete(final MutationProto proto
, final CellScanner cellScanner
)
587 MutationType type
= proto
.getMutateType();
588 assert type
== MutationType
.DELETE
: type
.name();
589 long timestamp
= proto
.hasTimestamp() ? proto
.getTimestamp() : HConstants
.LATEST_TIMESTAMP
;
590 Delete delete
= proto
.hasRow() ?
new Delete(proto
.getRow().toByteArray(), timestamp
) : null;
591 int cellCount
= proto
.hasAssociatedCellCount()? proto
.getAssociatedCellCount(): 0;
593 // The proto has metadata only and the data is separate to be found in the cellScanner.
594 if (cellScanner
== null) {
595 // TextFormat should be fine for a Delete since it carries no data, just coordinates.
596 throw new DoNotRetryIOException("Cell count of " + cellCount
+ " but no cellScanner: " +
597 TextFormat
.shortDebugString(proto
));
599 for (int i
= 0; i
< cellCount
; i
++) {
600 if (!cellScanner
.advance()) {
601 // TextFormat should be fine for a Delete since it carries no data, just coordinates.
602 throw new DoNotRetryIOException("Cell count of " + cellCount
+ " but at index " + i
+
603 " no cell returned: " + TextFormat
.shortDebugString(proto
));
605 Cell cell
= cellScanner
.current();
606 if (delete
== null) {
608 new Delete(cell
.getRowArray(), cell
.getRowOffset(), cell
.getRowLength(), timestamp
);
613 if (delete
== null) {
614 throw new IllegalArgumentException("row cannot be null");
616 for (ColumnValue column
: proto
.getColumnValueList()) {
617 byte[] family
= column
.getFamily().toByteArray();
618 for (QualifierValue qv
: column
.getQualifierValueList()) {
619 DeleteType deleteType
= qv
.getDeleteType();
620 byte[] qualifier
= null;
621 if (qv
.hasQualifier()) {
622 qualifier
= qv
.getQualifier().toByteArray();
624 long ts
= HConstants
.LATEST_TIMESTAMP
;
625 if (qv
.hasTimestamp()) {
626 ts
= qv
.getTimestamp();
628 if (deleteType
== DeleteType
.DELETE_ONE_VERSION
) {
629 delete
.addColumn(family
, qualifier
, ts
);
630 } else if (deleteType
== DeleteType
.DELETE_MULTIPLE_VERSIONS
) {
631 delete
.addColumns(family
, qualifier
, ts
);
632 } else if (deleteType
== DeleteType
.DELETE_FAMILY_VERSION
) {
633 delete
.addFamilyVersion(family
, ts
);
635 delete
.addFamily(family
, ts
);
640 delete
.setDurability(toDurability(proto
.getDurability()));
641 for (NameBytesPair attribute
: proto
.getAttributeList()) {
642 delete
.setAttribute(attribute
.getName(), attribute
.getValue().toByteArray());
648 private interface ConsumerWithException
<T
, U
> {
649 void accept(T t
, U u
) throws IOException
;
652 private static <T
extends Mutation
> T
toDelta(Function
<Bytes
, T
> supplier
, ConsumerWithException
<T
, Cell
> consumer
,
653 final MutationProto proto
, final CellScanner cellScanner
) throws IOException
{
654 byte[] row
= proto
.hasRow() ? proto
.getRow().toByteArray() : null;
655 T mutation
= row
== null ?
null : supplier
.apply(new Bytes(row
));
656 int cellCount
= proto
.hasAssociatedCellCount() ? proto
.getAssociatedCellCount() : 0;
658 // The proto has metadata only and the data is separate to be found in the cellScanner.
659 if (cellScanner
== null) {
660 throw new DoNotRetryIOException("Cell count of " + cellCount
+ " but no cellScanner: " +
661 toShortString(proto
));
663 for (int i
= 0; i
< cellCount
; i
++) {
664 if (!cellScanner
.advance()) {
665 throw new DoNotRetryIOException("Cell count of " + cellCount
+ " but at index " + i
+
666 " no cell returned: " + toShortString(proto
));
668 Cell cell
= cellScanner
.current();
669 if (mutation
== null) {
670 mutation
= supplier
.apply(new Bytes(cell
.getRowArray(), cell
.getRowOffset(), cell
.getRowLength()));
672 consumer
.accept(mutation
, cell
);
675 if (mutation
== null) {
676 throw new IllegalArgumentException("row cannot be null");
678 for (ColumnValue column
: proto
.getColumnValueList()) {
679 byte[] family
= column
.getFamily().toByteArray();
680 for (QualifierValue qv
: column
.getQualifierValueList()) {
681 byte[] qualifier
= qv
.getQualifier().toByteArray();
682 if (!qv
.hasValue()) {
683 throw new DoNotRetryIOException(
684 "Missing required field: qualifier value");
686 byte[] value
= qv
.getValue().toByteArray();
689 tags
= qv
.getTags().toByteArray();
691 consumer
.accept(mutation
, ExtendedCellBuilderFactory
.create(CellBuilderType
.DEEP_COPY
)
692 .setRow(mutation
.getRow()).setFamily(family
)
693 .setQualifier(qualifier
).setTimestamp(qv
.getTimestamp())
694 .setType(KeyValue
.Type
.Put
.getCode()).setValue(value
)
695 .setTags(tags
).setSequenceId(0)
700 mutation
.setDurability(toDurability(proto
.getDurability()));
701 for (NameBytesPair attribute
: proto
.getAttributeList()) {
702 mutation
.setAttribute(attribute
.getName(), attribute
.getValue().toByteArray());
708 * Convert a protocol buffer Mutate to an Append
710 * @param proto the protocol buffer Mutate to convert
711 * @return the converted client Append
712 * @throws IOException
714 public static Append
toAppend(final MutationProto proto
, final CellScanner cellScanner
)
716 MutationType type
= proto
.getMutateType();
717 assert type
== MutationType
.APPEND
: type
.name();
718 Append append
= toDelta((Bytes row
) -> new Append(row
.get(), row
.getOffset(), row
.getLength()),
719 Append
::add
, proto
, cellScanner
);
720 if (proto
.hasTimeRange()) {
721 TimeRange timeRange
= protoToTimeRange(proto
.getTimeRange());
722 append
.setTimeRange(timeRange
.getMin(), timeRange
.getMax());
728 * Convert a protocol buffer Mutate to an Increment
730 * @param proto the protocol buffer Mutate to convert
731 * @return the converted client Increment
732 * @throws IOException
734 public static Increment
toIncrement(final MutationProto proto
, final CellScanner cellScanner
)
736 MutationType type
= proto
.getMutateType();
737 assert type
== MutationType
.INCREMENT
: type
.name();
738 Increment increment
= toDelta((Bytes row
) -> new Increment(row
.get(), row
.getOffset(), row
.getLength()),
739 Increment
::add
, proto
, cellScanner
);
740 if (proto
.hasTimeRange()) {
741 TimeRange timeRange
= protoToTimeRange(proto
.getTimeRange());
742 increment
.setTimeRange(timeRange
.getMin(), timeRange
.getMax());
748 * Convert a MutateRequest to Mutation
750 * @param proto the protocol buffer Mutate to convert
751 * @return the converted Mutation
752 * @throws IOException
754 public static Mutation
toMutation(final MutationProto proto
) throws IOException
{
755 MutationType type
= proto
.getMutateType();
756 if (type
== MutationType
.APPEND
) {
757 return toAppend(proto
, null);
759 if (type
== MutationType
.DELETE
) {
760 return toDelete(proto
, null);
762 if (type
== MutationType
.PUT
) {
763 return toPut(proto
, null);
765 throw new IOException("Unknown mutation type " + type
);
769 * Convert a protocol buffer Mutate to a Get.
770 * @param proto the protocol buffer Mutate to convert.
772 * @return the converted client get.
773 * @throws IOException
775 public static Get
toGet(final MutationProto proto
, final CellScanner cellScanner
)
777 MutationType type
= proto
.getMutateType();
778 assert type
== MutationType
.INCREMENT
|| type
== MutationType
.APPEND
: type
.name();
779 byte[] row
= proto
.hasRow() ? proto
.getRow().toByteArray() : null;
781 int cellCount
= proto
.hasAssociatedCellCount() ? proto
.getAssociatedCellCount() : 0;
783 // The proto has metadata only and the data is separate to be found in the cellScanner.
784 if (cellScanner
== null) {
785 throw new DoNotRetryIOException("Cell count of " + cellCount
+ " but no cellScanner: "
786 + TextFormat
.shortDebugString(proto
));
788 for (int i
= 0; i
< cellCount
; i
++) {
789 if (!cellScanner
.advance()) {
790 throw new DoNotRetryIOException("Cell count of " + cellCount
+ " but at index " + i
791 + " no cell returned: " + TextFormat
.shortDebugString(proto
));
793 Cell cell
= cellScanner
.current();
795 get
= new Get(Bytes
.copy(cell
.getRowArray(), cell
.getRowOffset(), cell
.getRowLength()));
798 Bytes
.copy(cell
.getFamilyArray(), cell
.getFamilyOffset(), cell
.getFamilyLength()),
799 Bytes
.copy(cell
.getQualifierArray(), cell
.getQualifierOffset(),
800 cell
.getQualifierLength()));
804 for (ColumnValue column
: proto
.getColumnValueList()) {
805 byte[] family
= column
.getFamily().toByteArray();
806 for (QualifierValue qv
: column
.getQualifierValueList()) {
807 byte[] qualifier
= qv
.getQualifier().toByteArray();
808 if (!qv
.hasValue()) {
809 throw new DoNotRetryIOException("Missing required field: qualifier value");
811 get
.addColumn(family
, qualifier
);
815 if (proto
.hasTimeRange()) {
816 TimeRange timeRange
= protoToTimeRange(proto
.getTimeRange());
817 get
.setTimeRange(timeRange
.getMin(), timeRange
.getMax());
819 for (NameBytesPair attribute
: proto
.getAttributeList()) {
820 get
.setAttribute(attribute
.getName(), attribute
.getValue().toByteArray());
825 public static ClientProtos
.Scan
.ReadType
toReadType(Scan
.ReadType readType
) {
828 return ClientProtos
.Scan
.ReadType
.DEFAULT
;
830 return ClientProtos
.Scan
.ReadType
.STREAM
;
832 return ClientProtos
.Scan
.ReadType
.PREAD
;
834 throw new IllegalArgumentException("Unknown ReadType: " + readType
);
838 public static Scan
.ReadType
toReadType(ClientProtos
.Scan
.ReadType readType
) {
841 return Scan
.ReadType
.DEFAULT
;
843 return Scan
.ReadType
.STREAM
;
845 return Scan
.ReadType
.PREAD
;
847 throw new IllegalArgumentException("Unknown ReadType: " + readType
);
852 * Convert a client Scan to a protocol buffer Scan
854 * @param scan the client Scan to convert
855 * @return the converted protocol buffer Scan
856 * @throws IOException
858 public static ClientProtos
.Scan
toScan(
859 final Scan scan
) throws IOException
{
860 ClientProtos
.Scan
.Builder scanBuilder
=
861 ClientProtos
.Scan
.newBuilder();
862 scanBuilder
.setCacheBlocks(scan
.getCacheBlocks());
863 if (scan
.getBatch() > 0) {
864 scanBuilder
.setBatchSize(scan
.getBatch());
866 if (scan
.getMaxResultSize() > 0) {
867 scanBuilder
.setMaxResultSize(scan
.getMaxResultSize());
869 if (scan
.isSmall()) {
870 scanBuilder
.setSmall(scan
.isSmall());
872 if (scan
.getAllowPartialResults()) {
873 scanBuilder
.setAllowPartialResults(scan
.getAllowPartialResults());
875 Boolean loadColumnFamiliesOnDemand
= scan
.getLoadColumnFamiliesOnDemandValue();
876 if (loadColumnFamiliesOnDemand
!= null) {
877 scanBuilder
.setLoadColumnFamiliesOnDemand(loadColumnFamiliesOnDemand
);
879 scanBuilder
.setMaxVersions(scan
.getMaxVersions());
880 scan
.getColumnFamilyTimeRange().forEach((cf
, timeRange
) -> {
881 scanBuilder
.addCfTimeRange(HBaseProtos
.ColumnFamilyTimeRange
.newBuilder()
882 .setColumnFamily(ByteStringer
.wrap(cf
))
883 .setTimeRange(toTimeRange(timeRange
))
886 scanBuilder
.setTimeRange(toTimeRange(scan
.getTimeRange()));
887 Map
<String
, byte[]> attributes
= scan
.getAttributesMap();
888 if (!attributes
.isEmpty()) {
889 NameBytesPair
.Builder attributeBuilder
= NameBytesPair
.newBuilder();
890 for (Map
.Entry
<String
, byte[]> attribute
: attributes
.entrySet()) {
891 attributeBuilder
.setName(attribute
.getKey());
892 attributeBuilder
.setValue(ByteStringer
.wrap(attribute
.getValue()));
893 scanBuilder
.addAttribute(attributeBuilder
.build());
896 byte[] startRow
= scan
.getStartRow();
897 if (startRow
!= null && startRow
.length
> 0) {
898 scanBuilder
.setStartRow(ByteStringer
.wrap(startRow
));
900 byte[] stopRow
= scan
.getStopRow();
901 if (stopRow
!= null && stopRow
.length
> 0) {
902 scanBuilder
.setStopRow(ByteStringer
.wrap(stopRow
));
904 if (scan
.hasFilter()) {
905 scanBuilder
.setFilter(ProtobufUtil
.toFilter(scan
.getFilter()));
907 if (scan
.hasFamilies()) {
908 Column
.Builder columnBuilder
= Column
.newBuilder();
909 for (Map
.Entry
<byte[],NavigableSet
<byte []>>
910 family
: scan
.getFamilyMap().entrySet()) {
911 columnBuilder
.setFamily(ByteStringer
.wrap(family
.getKey()));
912 NavigableSet
<byte []> qualifiers
= family
.getValue();
913 columnBuilder
.clearQualifier();
914 if (qualifiers
!= null && qualifiers
.size() > 0) {
915 for (byte [] qualifier
: qualifiers
) {
916 columnBuilder
.addQualifier(ByteStringer
.wrap(qualifier
));
919 scanBuilder
.addColumn(columnBuilder
.build());
922 if (scan
.getMaxResultsPerColumnFamily() >= 0) {
923 scanBuilder
.setStoreLimit(scan
.getMaxResultsPerColumnFamily());
925 if (scan
.getRowOffsetPerColumnFamily() > 0) {
926 scanBuilder
.setStoreOffset(scan
.getRowOffsetPerColumnFamily());
928 if (scan
.isReversed()) {
929 scanBuilder
.setReversed(scan
.isReversed());
931 if (scan
.getConsistency() == Consistency
.TIMELINE
) {
932 scanBuilder
.setConsistency(toConsistency(scan
.getConsistency()));
934 if (scan
.getCaching() > 0) {
935 scanBuilder
.setCaching(scan
.getCaching());
937 long mvccReadPoint
= PackagePrivateFieldAccessor
.getMvccReadPoint(scan
);
938 if (mvccReadPoint
> 0) {
939 scanBuilder
.setMvccReadPoint(mvccReadPoint
);
941 if (!scan
.includeStartRow()) {
942 scanBuilder
.setIncludeStartRow(false);
944 scanBuilder
.setIncludeStopRow(scan
.includeStopRow());
945 if (scan
.getReadType() != Scan
.ReadType
.DEFAULT
) {
946 scanBuilder
.setReadType(toReadType(scan
.getReadType()));
948 return scanBuilder
.build();
952 * Convert a protocol buffer Scan to a client Scan
954 * @param proto the protocol buffer Scan to convert
955 * @return the converted client Scan
956 * @throws IOException
958 public static Scan
toScan(
959 final ClientProtos
.Scan proto
) throws IOException
{
960 byte[] startRow
= HConstants
.EMPTY_START_ROW
;
961 byte[] stopRow
= HConstants
.EMPTY_END_ROW
;
962 boolean includeStartRow
= true;
963 boolean includeStopRow
= false;
964 if (proto
.hasStartRow()) {
965 startRow
= proto
.getStartRow().toByteArray();
967 if (proto
.hasStopRow()) {
968 stopRow
= proto
.getStopRow().toByteArray();
970 if (proto
.hasIncludeStartRow()) {
971 includeStartRow
= proto
.getIncludeStartRow();
973 if (proto
.hasIncludeStopRow()) {
974 includeStopRow
= proto
.getIncludeStopRow();
977 new Scan().withStartRow(startRow
, includeStartRow
).withStopRow(stopRow
, includeStopRow
);
978 if (proto
.hasCacheBlocks()) {
979 scan
.setCacheBlocks(proto
.getCacheBlocks());
981 if (proto
.hasMaxVersions()) {
982 scan
.setMaxVersions(proto
.getMaxVersions());
984 if (proto
.hasStoreLimit()) {
985 scan
.setMaxResultsPerColumnFamily(proto
.getStoreLimit());
987 if (proto
.hasStoreOffset()) {
988 scan
.setRowOffsetPerColumnFamily(proto
.getStoreOffset());
990 if (proto
.hasLoadColumnFamiliesOnDemand()) {
991 scan
.setLoadColumnFamiliesOnDemand(proto
.getLoadColumnFamiliesOnDemand());
993 if (proto
.getCfTimeRangeCount() > 0) {
994 for (HBaseProtos
.ColumnFamilyTimeRange cftr
: proto
.getCfTimeRangeList()) {
995 TimeRange timeRange
= protoToTimeRange(cftr
.getTimeRange());
996 scan
.setColumnFamilyTimeRange(cftr
.getColumnFamily().toByteArray(),
997 timeRange
.getMin(), timeRange
.getMax());
1000 if (proto
.hasTimeRange()) {
1001 TimeRange timeRange
= protoToTimeRange(proto
.getTimeRange());
1002 scan
.setTimeRange(timeRange
.getMin(), timeRange
.getMax());
1004 if (proto
.hasFilter()) {
1005 FilterProtos
.Filter filter
= proto
.getFilter();
1006 scan
.setFilter(ProtobufUtil
.toFilter(filter
));
1008 if (proto
.hasBatchSize()) {
1009 scan
.setBatch(proto
.getBatchSize());
1011 if (proto
.hasMaxResultSize()) {
1012 scan
.setMaxResultSize(proto
.getMaxResultSize());
1014 if (proto
.hasSmall()) {
1015 scan
.setSmall(proto
.getSmall());
1017 if (proto
.hasAllowPartialResults()) {
1018 scan
.setAllowPartialResults(proto
.getAllowPartialResults());
1020 for (NameBytesPair attribute
: proto
.getAttributeList()) {
1021 scan
.setAttribute(attribute
.getName(), attribute
.getValue().toByteArray());
1023 if (proto
.getColumnCount() > 0) {
1024 for (Column column
: proto
.getColumnList()) {
1025 byte[] family
= column
.getFamily().toByteArray();
1026 if (column
.getQualifierCount() > 0) {
1027 for (ByteString qualifier
: column
.getQualifierList()) {
1028 scan
.addColumn(family
, qualifier
.toByteArray());
1031 scan
.addFamily(family
);
1035 if (proto
.hasReversed()) {
1036 scan
.setReversed(proto
.getReversed());
1038 if (proto
.hasConsistency()) {
1039 scan
.setConsistency(toConsistency(proto
.getConsistency()));
1041 if (proto
.hasCaching()) {
1042 scan
.setCaching(proto
.getCaching());
1044 if (proto
.hasMvccReadPoint()) {
1045 PackagePrivateFieldAccessor
.setMvccReadPoint(scan
, proto
.getMvccReadPoint());
1047 if (scan
.isSmall()) {
1048 scan
.setReadType(Scan
.ReadType
.PREAD
);
1049 } else if (proto
.hasReadType()) {
1050 scan
.setReadType(toReadType(proto
.getReadType()));
1056 * Create a protocol buffer Get based on a client Get.
1058 * @param get the client Get
1059 * @return a protocol buffer Get
1060 * @throws IOException
1062 public static ClientProtos
.Get
toGet(
1063 final Get get
) throws IOException
{
1064 ClientProtos
.Get
.Builder builder
=
1065 ClientProtos
.Get
.newBuilder();
1066 builder
.setRow(ByteStringer
.wrap(get
.getRow()));
1067 builder
.setCacheBlocks(get
.getCacheBlocks());
1068 builder
.setMaxVersions(get
.getMaxVersions());
1069 if (get
.getFilter() != null) {
1070 builder
.setFilter(ProtobufUtil
.toFilter(get
.getFilter()));
1072 get
.getColumnFamilyTimeRange().forEach((cf
, timeRange
) ->
1073 builder
.addCfTimeRange(HBaseProtos
.ColumnFamilyTimeRange
.newBuilder()
1074 .setColumnFamily(ByteStringer
.wrap(cf
))
1075 .setTimeRange(toTimeRange(timeRange
)).build())
1077 builder
.setTimeRange(toTimeRange(get
.getTimeRange()));
1078 Map
<String
, byte[]> attributes
= get
.getAttributesMap();
1079 if (!attributes
.isEmpty()) {
1080 NameBytesPair
.Builder attributeBuilder
= NameBytesPair
.newBuilder();
1081 for (Map
.Entry
<String
, byte[]> attribute
: attributes
.entrySet()) {
1082 attributeBuilder
.setName(attribute
.getKey());
1083 attributeBuilder
.setValue(ByteStringer
.wrap(attribute
.getValue()));
1084 builder
.addAttribute(attributeBuilder
.build());
1087 if (get
.hasFamilies()) {
1088 Column
.Builder columnBuilder
= Column
.newBuilder();
1089 Map
<byte[], NavigableSet
<byte[]>> families
= get
.getFamilyMap();
1090 for (Map
.Entry
<byte[], NavigableSet
<byte[]>> family
: families
.entrySet()) {
1091 NavigableSet
<byte[]> qualifiers
= family
.getValue();
1092 columnBuilder
.setFamily(ByteStringer
.wrap(family
.getKey()));
1093 columnBuilder
.clearQualifier();
1094 if (qualifiers
!= null && qualifiers
.size() > 0) {
1095 for (byte[] qualifier
: qualifiers
) {
1096 columnBuilder
.addQualifier(ByteStringer
.wrap(qualifier
));
1099 builder
.addColumn(columnBuilder
.build());
1102 if (get
.getMaxResultsPerColumnFamily() >= 0) {
1103 builder
.setStoreLimit(get
.getMaxResultsPerColumnFamily());
1105 if (get
.getRowOffsetPerColumnFamily() > 0) {
1106 builder
.setStoreOffset(get
.getRowOffsetPerColumnFamily());
1108 if (get
.isCheckExistenceOnly()){
1109 builder
.setExistenceOnly(true);
1111 if (get
.getConsistency() != null && get
.getConsistency() != Consistency
.STRONG
) {
1112 builder
.setConsistency(toConsistency(get
.getConsistency()));
1115 Boolean loadColumnFamiliesOnDemand
= get
.getLoadColumnFamiliesOnDemandValue();
1116 if (loadColumnFamiliesOnDemand
!= null) {
1117 builder
.setLoadColumnFamiliesOnDemand(loadColumnFamiliesOnDemand
);
1120 return builder
.build();
1123 public static MutationProto
toMutation(final MutationType type
, final Mutation mutation
)
1124 throws IOException
{
1125 return toMutation(type
, mutation
, HConstants
.NO_NONCE
);
1129 * Create a protocol buffer Mutate based on a client Mutation
1133 * @return a protobuf'd Mutation
1134 * @throws IOException
1136 public static MutationProto
toMutation(final MutationType type
, final Mutation mutation
,
1137 final long nonce
) throws IOException
{
1138 return toMutation(type
, mutation
, MutationProto
.newBuilder(), nonce
);
1141 public static MutationProto
toMutation(final MutationType type
, final Mutation mutation
,
1142 MutationProto
.Builder builder
) throws IOException
{
1143 return toMutation(type
, mutation
, builder
, HConstants
.NO_NONCE
);
1146 public static MutationProto
toMutation(final MutationType type
, final Mutation mutation
,
1147 MutationProto
.Builder builder
, long nonce
)
1148 throws IOException
{
1149 builder
= getMutationBuilderAndSetCommonFields(type
, mutation
, builder
);
1150 if (nonce
!= HConstants
.NO_NONCE
) {
1151 builder
.setNonce(nonce
);
1153 if (type
== MutationType
.INCREMENT
) {
1154 builder
.setTimeRange(toTimeRange(((Increment
) mutation
).getTimeRange()));
1156 if (type
== MutationType
.APPEND
) {
1157 builder
.setTimeRange(toTimeRange(((Append
) mutation
).getTimeRange()));
1159 ColumnValue
.Builder columnBuilder
= ColumnValue
.newBuilder();
1160 QualifierValue
.Builder valueBuilder
= QualifierValue
.newBuilder();
1161 for (Map
.Entry
<byte[],List
<Cell
>> family
: mutation
.getFamilyCellMap().entrySet()) {
1162 columnBuilder
.clear();
1163 columnBuilder
.setFamily(ByteStringer
.wrap(family
.getKey()));
1164 for (Cell cell
: family
.getValue()) {
1165 valueBuilder
.clear();
1166 valueBuilder
.setQualifier(ByteStringer
.wrap(
1167 cell
.getQualifierArray(), cell
.getQualifierOffset(), cell
.getQualifierLength()));
1168 valueBuilder
.setValue(ByteStringer
.wrap(
1169 cell
.getValueArray(), cell
.getValueOffset(), cell
.getValueLength()));
1170 valueBuilder
.setTimestamp(cell
.getTimestamp());
1171 if (type
== MutationType
.DELETE
|| (type
== MutationType
.PUT
&& CellUtil
.isDelete(cell
))) {
1172 KeyValue
.Type keyValueType
= KeyValue
.Type
.codeToType(cell
.getTypeByte());
1173 valueBuilder
.setDeleteType(toDeleteType(keyValueType
));
1175 columnBuilder
.addQualifierValue(valueBuilder
.build());
1177 builder
.addColumnValue(columnBuilder
.build());
1179 return builder
.build();
1183 * Create a protocol buffer MutationProto based on a client Mutation. Does NOT include data.
1184 * Understanding is that the Cell will be transported other than via protobuf.
1188 * @return a protobuf'd Mutation
1189 * @throws IOException
1191 public static MutationProto
toMutationNoData(final MutationType type
, final Mutation mutation
,
1192 final MutationProto
.Builder builder
) throws IOException
{
1193 return toMutationNoData(type
, mutation
, builder
, HConstants
.NO_NONCE
);
1197 * Create a protocol buffer MutationProto based on a client Mutation. Does NOT include data.
1198 * Understanding is that the Cell will be transported other than via protobuf.
1201 * @return a protobuf'd Mutation
1202 * @throws IOException
1204 public static MutationProto
toMutationNoData(final MutationType type
, final Mutation mutation
)
1205 throws IOException
{
1206 MutationProto
.Builder builder
= MutationProto
.newBuilder();
1207 return toMutationNoData(type
, mutation
, builder
);
1210 public static MutationProto
toMutationNoData(final MutationType type
, final Mutation mutation
,
1211 final MutationProto
.Builder builder
, long nonce
) throws IOException
{
1212 getMutationBuilderAndSetCommonFields(type
, mutation
, builder
);
1213 builder
.setAssociatedCellCount(mutation
.size());
1214 if (mutation
instanceof Increment
) {
1215 builder
.setTimeRange(toTimeRange(((Increment
)mutation
).getTimeRange()));
1217 if (mutation
instanceof Append
) {
1218 builder
.setTimeRange(toTimeRange(((Append
)mutation
).getTimeRange()));
1220 if (nonce
!= HConstants
.NO_NONCE
) {
1221 builder
.setNonce(nonce
);
1223 return builder
.build();
1227 * Code shared by {@link #toMutation(MutationType, Mutation)} and
1228 * {@link #toMutationNoData(MutationType, Mutation)}
1231 * @return A partly-filled out protobuf'd Mutation.
1233 private static MutationProto
.Builder
getMutationBuilderAndSetCommonFields(final MutationType type
,
1234 final Mutation mutation
, MutationProto
.Builder builder
) {
1235 builder
.setRow(ByteStringer
.wrap(mutation
.getRow()));
1236 builder
.setMutateType(type
);
1237 builder
.setDurability(toDurability(mutation
.getDurability()));
1238 builder
.setTimestamp(mutation
.getTimestamp());
1239 Map
<String
, byte[]> attributes
= mutation
.getAttributesMap();
1240 if (!attributes
.isEmpty()) {
1241 NameBytesPair
.Builder attributeBuilder
= NameBytesPair
.newBuilder();
1242 for (Map
.Entry
<String
, byte[]> attribute
: attributes
.entrySet()) {
1243 attributeBuilder
.setName(attribute
.getKey());
1244 attributeBuilder
.setValue(ByteStringer
.wrap(attribute
.getValue()));
1245 builder
.addAttribute(attributeBuilder
.build());
1252 * Convert a client Result to a protocol buffer Result
1254 * @param result the client Result to convert
1255 * @return the converted protocol buffer Result
1257 public static ClientProtos
.Result
toResult(final Result result
) {
1258 if (result
.getExists() != null) {
1259 return toResult(result
.getExists(), result
.isStale());
1262 Cell
[] cells
= result
.rawCells();
1263 if (cells
== null || cells
.length
== 0) {
1264 return result
.isStale() ? EMPTY_RESULT_PB_STALE
: EMPTY_RESULT_PB
;
1267 ClientProtos
.Result
.Builder builder
= ClientProtos
.Result
.newBuilder();
1268 for (Cell c
: cells
) {
1269 builder
.addCell(toCell(c
));
1272 builder
.setStale(result
.isStale());
1273 builder
.setPartial(result
.mayHaveMoreCellsInRow());
1275 return builder
.build();
1279 * Convert a client Result to a protocol buffer Result
1281 * @param existence the client existence to send
1282 * @return the converted protocol buffer Result
1284 public static ClientProtos
.Result
toResult(final boolean existence
, boolean stale
) {
1286 return existence ? EMPTY_RESULT_PB_EXISTS_TRUE_STALE
: EMPTY_RESULT_PB_EXISTS_FALSE_STALE
;
1288 return existence ? EMPTY_RESULT_PB_EXISTS_TRUE
: EMPTY_RESULT_PB_EXISTS_FALSE
;
1293 * Convert a client Result to a protocol buffer Result.
1294 * The pb Result does not include the Cell data. That is for transport otherwise.
1296 * @param result the client Result to convert
1297 * @return the converted protocol buffer Result
1299 public static ClientProtos
.Result
toResultNoData(final Result result
) {
1300 if (result
.getExists() != null) return toResult(result
.getExists(), result
.isStale());
1301 int size
= result
.size();
1302 if (size
== 0) return result
.isStale() ? EMPTY_RESULT_PB_STALE
: EMPTY_RESULT_PB
;
1303 ClientProtos
.Result
.Builder builder
= ClientProtos
.Result
.newBuilder();
1304 builder
.setAssociatedCellCount(size
);
1305 builder
.setStale(result
.isStale());
1306 return builder
.build();
1310 * Convert a protocol buffer Result to a client Result
1312 * @param proto the protocol buffer Result to convert
1313 * @return the converted client Result
1315 public static Result
toResult(final ClientProtos
.Result proto
) {
1316 if (proto
.hasExists()) {
1317 if (proto
.getStale()) {
1318 return proto
.getExists() ? EMPTY_RESULT_EXISTS_TRUE_STALE
:EMPTY_RESULT_EXISTS_FALSE_STALE
;
1320 return proto
.getExists() ? EMPTY_RESULT_EXISTS_TRUE
: EMPTY_RESULT_EXISTS_FALSE
;
1323 List
<CellProtos
.Cell
> values
= proto
.getCellList();
1324 if (values
.isEmpty()){
1325 return proto
.getStale() ? EMPTY_RESULT_STALE
: EMPTY_RESULT
;
1328 List
<Cell
> cells
= new ArrayList
<>(values
.size());
1329 ExtendedCellBuilder builder
= ExtendedCellBuilderFactory
.create(CellBuilderType
.SHALLOW_COPY
);
1330 for (CellProtos
.Cell c
: values
) {
1331 cells
.add(toCell(builder
, c
));
1333 return Result
.create(cells
, null, proto
.getStale(), proto
.getPartial());
1337 * Convert a protocol buffer Result to a client Result
1339 * @param proto the protocol buffer Result to convert
1340 * @param scanner Optional cell scanner.
1341 * @return the converted client Result
1342 * @throws IOException
1344 public static Result
toResult(final ClientProtos
.Result proto
, final CellScanner scanner
)
1345 throws IOException
{
1346 List
<CellProtos
.Cell
> values
= proto
.getCellList();
1348 if (proto
.hasExists()) {
1349 if (!values
.isEmpty() ||
1350 (proto
.hasAssociatedCellCount() && proto
.getAssociatedCellCount() > 0)) {
1351 throw new IllegalArgumentException("bad proto: exists with cells is no allowed " + proto
);
1353 if (proto
.getStale()) {
1354 return proto
.getExists() ? EMPTY_RESULT_EXISTS_TRUE_STALE
:EMPTY_RESULT_EXISTS_FALSE_STALE
;
1356 return proto
.getExists() ? EMPTY_RESULT_EXISTS_TRUE
: EMPTY_RESULT_EXISTS_FALSE
;
1359 // TODO: Unit test that has some Cells in scanner and some in the proto.
1360 List
<Cell
> cells
= null;
1361 if (proto
.hasAssociatedCellCount()) {
1362 int count
= proto
.getAssociatedCellCount();
1363 cells
= new ArrayList
<>(count
+ values
.size());
1364 for (int i
= 0; i
< count
; i
++) {
1365 if (!scanner
.advance()) throw new IOException("Failed get " + i
+ " of " + count
);
1366 cells
.add(scanner
.current());
1370 if (!values
.isEmpty()){
1371 if (cells
== null) cells
= new ArrayList
<>(values
.size());
1372 ExtendedCellBuilder builder
= ExtendedCellBuilderFactory
.create(CellBuilderType
.SHALLOW_COPY
);
1373 for (CellProtos
.Cell c
: values
) {
1374 cells
.add(toCell(builder
, c
));
1378 return (cells
== null || cells
.isEmpty())
1379 ?
(proto
.getStale() ? EMPTY_RESULT_STALE
: EMPTY_RESULT
)
1380 : Result
.create(cells
, null, proto
.getStale());
1385 * Convert a ByteArrayComparable to a protocol buffer Comparator
1387 * @param comparator the ByteArrayComparable to convert
1388 * @return the converted protocol buffer Comparator
1390 public static ComparatorProtos
.Comparator
toComparator(ByteArrayComparable comparator
) {
1391 ComparatorProtos
.Comparator
.Builder builder
= ComparatorProtos
.Comparator
.newBuilder();
1392 builder
.setName(comparator
.getClass().getName());
1393 builder
.setSerializedComparator(ByteStringer
.wrap(comparator
.toByteArray()));
1394 return builder
.build();
1398 * Convert a protocol buffer Comparator to a ByteArrayComparable
1400 * @param proto the protocol buffer Comparator to convert
1401 * @return the converted ByteArrayComparable
1403 @SuppressWarnings("unchecked")
1404 public static ByteArrayComparable
toComparator(ComparatorProtos
.Comparator proto
)
1405 throws IOException
{
1406 String type
= proto
.getName();
1407 String funcName
= "parseFrom";
1408 byte [] value
= proto
.getSerializedComparator().toByteArray();
1410 Class
<?
> c
= Class
.forName(type
, true, ClassLoaderHolder
.CLASS_LOADER
);
1411 Method parseFrom
= c
.getMethod(funcName
, byte[].class);
1412 if (parseFrom
== null) {
1413 throw new IOException("Unable to locate function: " + funcName
+ " in type: " + type
);
1415 return (ByteArrayComparable
)parseFrom
.invoke(null, value
);
1416 } catch (Exception e
) {
1417 throw new IOException(e
);
1422 * Convert a protocol buffer Filter to a client Filter
1424 * @param proto the protocol buffer Filter to convert
1425 * @return the converted Filter
1427 @SuppressWarnings("unchecked")
1428 public static Filter
toFilter(FilterProtos
.Filter proto
) throws IOException
{
1429 String type
= proto
.getName();
1430 final byte [] value
= proto
.getSerializedFilter().toByteArray();
1431 String funcName
= "parseFrom";
1433 Class
<?
> c
= Class
.forName(type
, true, ClassLoaderHolder
.CLASS_LOADER
);
1434 Method parseFrom
= c
.getMethod(funcName
, byte[].class);
1435 if (parseFrom
== null) {
1436 throw new IOException("Unable to locate function: " + funcName
+ " in type: " + type
);
1438 return (Filter
)parseFrom
.invoke(c
, value
);
1439 } catch (Exception e
) {
1440 // Either we couldn't instantiate the method object, or "parseFrom" failed.
1441 // In either case, let's not retry.
1442 throw new DoNotRetryIOException(e
);
1447 * Convert a client Filter to a protocol buffer Filter
1449 * @param filter the Filter to convert
1450 * @return the converted protocol buffer Filter
1452 public static FilterProtos
.Filter
toFilter(Filter filter
) throws IOException
{
1453 FilterProtos
.Filter
.Builder builder
= FilterProtos
.Filter
.newBuilder();
1454 builder
.setName(filter
.getClass().getName());
1455 builder
.setSerializedFilter(ByteStringer
.wrap(filter
.toByteArray()));
1456 return builder
.build();
1460 * Convert a delete KeyValue type to protocol buffer DeleteType.
1463 * @return protocol buffer DeleteType
1464 * @throws IOException
1466 public static DeleteType
toDeleteType(
1467 KeyValue
.Type type
) throws IOException
{
1470 return DeleteType
.DELETE_ONE_VERSION
;
1472 return DeleteType
.DELETE_MULTIPLE_VERSIONS
;
1474 return DeleteType
.DELETE_FAMILY
;
1475 case DeleteFamilyVersion
:
1476 return DeleteType
.DELETE_FAMILY_VERSION
;
1478 throw new IOException("Unknown delete type: " + type
);
1483 * Convert a protocol buffer DeleteType to delete KeyValue type.
1485 * @param type The DeleteType
1487 * @throws IOException
1489 public static KeyValue
.Type
fromDeleteType(
1490 DeleteType type
) throws IOException
{
1492 case DELETE_ONE_VERSION
:
1493 return KeyValue
.Type
.Delete
;
1494 case DELETE_MULTIPLE_VERSIONS
:
1495 return KeyValue
.Type
.DeleteColumn
;
1497 return KeyValue
.Type
.DeleteFamily
;
1498 case DELETE_FAMILY_VERSION
:
1499 return KeyValue
.Type
.DeleteFamilyVersion
;
1501 throw new IOException("Unknown delete type: " + type
);
1506 * Convert a stringified protocol buffer exception Parameter to a Java Exception
1508 * @param parameter the protocol buffer Parameter to convert
1509 * @return the converted Exception
1510 * @throws IOException if failed to deserialize the parameter
1512 @SuppressWarnings("unchecked")
1513 public static Throwable
toException(final NameBytesPair parameter
) throws IOException
{
1514 if (parameter
== null || !parameter
.hasValue()) return null;
1515 String desc
= parameter
.getValue().toStringUtf8();
1516 String type
= parameter
.getName();
1518 Class
<?
extends Throwable
> c
=
1519 (Class
<?
extends Throwable
>)Class
.forName(type
, true, ClassLoaderHolder
.CLASS_LOADER
);
1520 Constructor
<?
extends Throwable
> cn
= null;
1522 cn
= c
.getDeclaredConstructor(String
.class);
1523 return cn
.newInstance(desc
);
1524 } catch (NoSuchMethodException e
) {
1525 // Could be a raw RemoteException. See HBASE-8987.
1526 cn
= c
.getDeclaredConstructor(String
.class, String
.class);
1527 return cn
.newInstance(type
, desc
);
1529 } catch (Exception e
) {
1530 throw new IOException(e
);
1534 // Start helpers for Client
1536 @SuppressWarnings("unchecked")
1537 public static <T
extends Service
> T
newServiceStub(Class
<T
> service
, RpcChannel channel
)
1539 return (T
)Methods
.call(service
, null, "newStub",
1540 new Class
[]{ RpcChannel
.class }, new Object
[]{ channel
});
1543 // End helpers for Client
1544 // Start helpers for Admin
1547 * A helper to get the info of a region server using admin protocol.
1548 * @return the server name
1550 public static ServerInfo
getServerInfo(final RpcController controller
,
1551 final AdminService
.BlockingInterface admin
)
1552 throws IOException
{
1553 GetServerInfoRequest request
= buildGetServerInfoRequest();
1555 GetServerInfoResponse response
= admin
.getServerInfo(controller
, request
);
1556 return response
.getServerInfo();
1557 } catch (ServiceException se
) {
1558 throw getRemoteException(se
);
1564 * @see #buildGetServerInfoRequest()
1566 private static GetServerInfoRequest GET_SERVER_INFO_REQUEST
=
1567 GetServerInfoRequest
.newBuilder().build();
1570 * Create a new GetServerInfoRequest
1572 * @return a GetServerInfoRequest
1574 public static GetServerInfoRequest
buildGetServerInfoRequest() {
1575 return GET_SERVER_INFO_REQUEST
;
1578 public static ScanMetrics
toScanMetrics(final byte[] bytes
) {
1579 Parser
<MapReduceProtos
.ScanMetrics
> parser
= MapReduceProtos
.ScanMetrics
.PARSER
;
1580 MapReduceProtos
.ScanMetrics pScanMetrics
= null;
1582 pScanMetrics
= parser
.parseFrom(bytes
);
1583 } catch (InvalidProtocolBufferException e
) {
1584 //Ignored there are just no key values to add.
1586 ScanMetrics scanMetrics
= new ScanMetrics();
1587 if (pScanMetrics
!= null) {
1588 for (HBaseProtos
.NameInt64Pair pair
: pScanMetrics
.getMetricsList()) {
1589 if (pair
.hasName() && pair
.hasValue()) {
1590 scanMetrics
.setCounter(pair
.getName(), pair
.getValue());
1598 * Unwraps an exception from a protobuf service into the underlying (expected) IOException. This
1599 * method will <strong>always</strong> throw an exception.
1600 * @param se the {@code ServiceException} instance to convert into an {@code IOException}
1601 * @throws NullPointerException if {@code se} is {@code null}
1603 public static void toIOException(ServiceException se
) throws IOException
{
1604 Objects
.requireNonNull(se
, "Service exception cannot be null");
1606 Throwable cause
= se
.getCause();
1607 if (cause
!= null && cause
instanceof IOException
) {
1608 throw (IOException
)cause
;
1610 throw new IOException(se
);
1613 public static CellProtos
.Cell
toCell(final Cell kv
) {
1614 // Doing this is going to kill us if we do it for all data passed.
1616 CellProtos
.Cell
.Builder kvbuilder
= CellProtos
.Cell
.newBuilder();
1617 kvbuilder
.setRow(ByteStringer
.wrap(kv
.getRowArray(), kv
.getRowOffset(),
1618 kv
.getRowLength()));
1619 kvbuilder
.setFamily(ByteStringer
.wrap(kv
.getFamilyArray(),
1620 kv
.getFamilyOffset(), kv
.getFamilyLength()));
1621 kvbuilder
.setQualifier(ByteStringer
.wrap(kv
.getQualifierArray(),
1622 kv
.getQualifierOffset(), kv
.getQualifierLength()));
1623 kvbuilder
.setCellType(CellProtos
.CellType
.valueOf(kv
.getTypeByte()));
1624 kvbuilder
.setTimestamp(kv
.getTimestamp());
1625 kvbuilder
.setValue(ByteStringer
.wrap(kv
.getValueArray(), kv
.getValueOffset(),
1626 kv
.getValueLength()));
1627 return kvbuilder
.build();
1630 public static Cell
toCell(ExtendedCellBuilder cellBuilder
, final CellProtos
.Cell cell
) {
1631 return cellBuilder
.clear()
1632 .setRow(cell
.getRow().toByteArray())
1633 .setFamily(cell
.getFamily().toByteArray())
1634 .setQualifier(cell
.getQualifier().toByteArray())
1635 .setTimestamp(cell
.getTimestamp())
1636 .setType((byte) cell
.getCellType().getNumber())
1637 .setValue(cell
.getValue().toByteArray())
1642 * Print out some subset of a MutationProto rather than all of it and its data
1643 * @param proto Protobuf to print out
1644 * @return Short String of mutation proto
1646 static String
toShortString(final MutationProto proto
) {
1647 return "row=" + Bytes
.toString(proto
.getRow().toByteArray()) +
1648 ", type=" + proto
.getMutateType().toString();
1651 public static TableName
toTableName(HBaseProtos
.TableName tableNamePB
) {
1652 return TableName
.valueOf(tableNamePB
.getNamespace().asReadOnlyByteBuffer(),
1653 tableNamePB
.getQualifier().asReadOnlyByteBuffer());
1656 public static HBaseProtos
.TableName
toProtoTableName(TableName tableName
) {
1657 return HBaseProtos
.TableName
.newBuilder()
1658 .setNamespace(ByteStringer
.wrap(tableName
.getNamespace()))
1659 .setQualifier(ByteStringer
.wrap(tableName
.getQualifier())).build();
1663 * This version of protobuf's mergeFrom avoids the hard-coded 64MB limit for decoding
1664 * buffers when working with byte arrays
1665 * @param builder current message builder
1666 * @param b byte array
1667 * @throws IOException
1669 public static void mergeFrom(Message
.Builder builder
, byte[] b
) throws IOException
{
1670 final CodedInputStream codedInput
= CodedInputStream
.newInstance(b
);
1671 codedInput
.setSizeLimit(b
.length
);
1672 builder
.mergeFrom(codedInput
);
1673 codedInput
.checkLastTagWas(0);
1677 * This version of protobuf's mergeFrom avoids the hard-coded 64MB limit for decoding
1678 * buffers when working with byte arrays
1679 * @param builder current message builder
1680 * @param b byte array
1683 * @throws IOException
1685 public static void mergeFrom(Message
.Builder builder
, byte[] b
, int offset
, int length
)
1686 throws IOException
{
1687 final CodedInputStream codedInput
= CodedInputStream
.newInstance(b
, offset
, length
);
1688 codedInput
.setSizeLimit(length
);
1689 builder
.mergeFrom(codedInput
);
1690 codedInput
.checkLastTagWas(0);
1693 private static TimeRange
protoToTimeRange(HBaseProtos
.TimeRange timeRange
) throws IOException
{
1695 long maxStamp
= Long
.MAX_VALUE
;
1696 if (timeRange
.hasFrom()) {
1697 minStamp
= timeRange
.getFrom();
1699 if (timeRange
.hasTo()) {
1700 maxStamp
= timeRange
.getTo();
1702 return new TimeRange(minStamp
, maxStamp
);
1706 * Creates {@link org.apache.hadoop.hbase.protobuf.generated.HBaseProtos.SnapshotDescription.Type}
1707 * from {@link SnapshotType}
1708 * @param type the SnapshotDescription type
1709 * @return the protobuf SnapshotDescription type
1711 public static HBaseProtos
.SnapshotDescription
.Type
1712 createProtosSnapShotDescType(SnapshotType type
) {
1713 return HBaseProtos
.SnapshotDescription
.Type
.valueOf(type
.name());
1717 * Convert a byte array to a protocol buffer RegionSpecifier
1719 * @param type the region specifier type
1720 * @param value the region specifier byte array value
1721 * @return a protocol buffer RegionSpecifier
1723 public static RegionSpecifier
buildRegionSpecifier(
1724 final RegionSpecifierType type
, final byte[] value
) {
1725 RegionSpecifier
.Builder regionBuilder
= RegionSpecifier
.newBuilder();
1726 regionBuilder
.setValue(ByteStringer
.wrap(value
));
1727 regionBuilder
.setType(type
);
1728 return regionBuilder
.build();
1732 * Get a ServerName from the passed in data bytes.
1733 * @param data Data with a serialize server name in it; can handle the old style
1734 * servername where servername was host and port. Works too with data that
1735 * begins w/ the pb 'PBUF' magic and that is then followed by a protobuf that
1736 * has a serialized {@link ServerName} in it.
1737 * @return Returns null if <code>data</code> is null else converts passed data
1738 * to a ServerName instance.
1739 * @throws DeserializationException
1741 public static ServerName
toServerName(final byte [] data
) throws DeserializationException
{
1742 if (data
== null || data
.length
<= 0) return null;
1743 if (ProtobufMagic
.isPBMagicPrefix(data
)) {
1744 int prefixLen
= ProtobufMagic
.lengthOfPBMagic();
1746 ZooKeeperProtos
.Master rss
=
1747 ZooKeeperProtos
.Master
.PARSER
.parseFrom(data
, prefixLen
, data
.length
- prefixLen
);
1748 org
.apache
.hadoop
.hbase
.protobuf
.generated
.HBaseProtos
.ServerName sn
=
1750 return ServerName
.valueOf(sn
.getHostName(), sn
.getPort(), sn
.getStartCode());
1751 } catch (/*InvalidProtocolBufferException*/IOException e
) {
1752 // A failed parse of the znode is pretty catastrophic. Rather than loop
1753 // retrying hoping the bad bytes will changes, and rather than change
1754 // the signature on this method to add an IOE which will send ripples all
1755 // over the code base, throw a RuntimeException. This should "never" happen.
1756 // Fail fast if it does.
1757 throw new DeserializationException(e
);
1760 // The str returned could be old style -- pre hbase-1502 -- which was
1761 // hostname and port seperated by a colon rather than hostname, port and
1762 // startcode delimited by a ','.
1763 String str
= Bytes
.toString(data
);
1764 int index
= str
.indexOf(ServerName
.SERVERNAME_SEPARATOR
);
1766 // Presume its ServerName serialized with versioned bytes.
1767 return ServerName
.parseVersionedServerName(data
);
1769 // Presume it a hostname:port format.
1770 String hostname
= Addressing
.parseHostname(str
);
1771 int port
= Addressing
.parsePort(str
);
1772 return ServerName
.valueOf(hostname
, port
, -1L);
1775 public static HBaseProtos
.TimeRange
toTimeRange(TimeRange timeRange
) {
1776 if (timeRange
== null) {
1777 timeRange
= TimeRange
.allTime();
1779 return HBaseProtos
.TimeRange
.newBuilder().setFrom(timeRange
.getMin()).setTo(timeRange
.getMax())
1783 public static RSGroupInfo
toGroupInfo(RSGroupProtos
.RSGroupInfo proto
) {
1784 RSGroupInfo RSGroupInfo
= new RSGroupInfo(proto
.getName());
1785 for (HBaseProtos
.ServerName el
: proto
.getServersList()) {
1786 RSGroupInfo
.addServer(Address
.fromParts(el
.getHostName(), el
.getPort()));
1788 for (HBaseProtos
.TableName pTableName
: proto
.getTablesList()) {
1789 RSGroupInfo
.addTable(ProtobufUtil
.toTableName(pTableName
));
1794 public static RSGroupProtos
.RSGroupInfo
toProtoGroupInfo(RSGroupInfo pojo
) {
1795 List
<HBaseProtos
.TableName
> tables
= new ArrayList
<>(pojo
.getTables().size());
1796 for (TableName arg
: pojo
.getTables()) {
1797 tables
.add(ProtobufUtil
.toProtoTableName(arg
));
1799 List
<HBaseProtos
.ServerName
> hostports
= new ArrayList
<>(pojo
.getServers().size());
1800 for (Address el
: pojo
.getServers()) {
1801 hostports
.add(HBaseProtos
.ServerName
.newBuilder().setHostName(el
.getHostname())
1802 .setPort(el
.getPort()).build());
1804 return RSGroupProtos
.RSGroupInfo
.newBuilder().setName(pojo
.getName()).addAllServers(hostports
)
1805 .addAllTables(tables
).build();