001/**
002 *
003 * Licensed to the Apache Software Foundation (ASF) under one
004 * or more contributor license agreements.  See the NOTICE file
005 * distributed with this work for additional information
006 * regarding copyright ownership.  The ASF licenses this file
007 * to you under the Apache License, Version 2.0 (the
008 * "License"); you may not use this file except in compliance
009 * with the License.  You may obtain a copy of the License at
010 *
011 *     http://www.apache.org/licenses/LICENSE-2.0
012 *
013 * Unless required by applicable law or agreed to in writing, software
014 * distributed under the License is distributed on an "AS IS" BASIS,
015 * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
016 * See the License for the specific language governing permissions and
017 * limitations under the License.
018 */
019package org.apache.hadoop.hbase.thrift2;
020
021import static org.apache.hadoop.hbase.util.Bytes.getBytes;
022
023import java.io.IOException;
024import java.nio.ByteBuffer;
025import java.util.ArrayList;
026import java.util.List;
027import java.util.Map;
028
029import org.apache.hadoop.hbase.Cell;
030import org.apache.hadoop.hbase.CellBuilderFactory;
031import org.apache.hadoop.hbase.CellBuilderType;
032import org.apache.hadoop.hbase.CellUtil;
033import org.apache.hadoop.hbase.CompareOperator;
034import org.apache.hadoop.hbase.HConstants;
035import org.apache.hadoop.hbase.HRegionInfo;
036import org.apache.hadoop.hbase.HRegionLocation;
037import org.apache.hadoop.hbase.PrivateCellUtil;
038import org.apache.hadoop.hbase.ServerName;
039import org.apache.hadoop.hbase.client.Append;
040import org.apache.hadoop.hbase.client.Delete;
041import org.apache.hadoop.hbase.client.Durability;
042import org.apache.hadoop.hbase.client.Get;
043import org.apache.hadoop.hbase.client.Increment;
044import org.apache.hadoop.hbase.client.OperationWithAttributes;
045import org.apache.hadoop.hbase.client.Put;
046import org.apache.hadoop.hbase.client.Result;
047import org.apache.hadoop.hbase.client.RowMutations;
048import org.apache.hadoop.hbase.client.Scan;
049import org.apache.hadoop.hbase.client.Scan.ReadType;
050import org.apache.hadoop.hbase.filter.ParseFilter;
051import org.apache.hadoop.hbase.security.visibility.Authorizations;
052import org.apache.hadoop.hbase.security.visibility.CellVisibility;
053import org.apache.hadoop.hbase.thrift2.generated.TAppend;
054import org.apache.hadoop.hbase.thrift2.generated.TColumn;
055import org.apache.hadoop.hbase.thrift2.generated.TColumnIncrement;
056import org.apache.hadoop.hbase.thrift2.generated.TColumnValue;
057import org.apache.hadoop.hbase.thrift2.generated.TCompareOp;
058import org.apache.hadoop.hbase.thrift2.generated.TDelete;
059import org.apache.hadoop.hbase.thrift2.generated.TDurability;
060import org.apache.hadoop.hbase.thrift2.generated.TGet;
061import org.apache.hadoop.hbase.thrift2.generated.THRegionInfo;
062import org.apache.hadoop.hbase.thrift2.generated.THRegionLocation;
063import org.apache.hadoop.hbase.thrift2.generated.TIncrement;
064import org.apache.hadoop.hbase.thrift2.generated.TMutation;
065import org.apache.hadoop.hbase.thrift2.generated.TPut;
066import org.apache.hadoop.hbase.thrift2.generated.TReadType;
067import org.apache.hadoop.hbase.thrift2.generated.TResult;
068import org.apache.hadoop.hbase.thrift2.generated.TRowMutations;
069import org.apache.hadoop.hbase.thrift2.generated.TScan;
070import org.apache.hadoop.hbase.thrift2.generated.TServerName;
071import org.apache.hadoop.hbase.thrift2.generated.TTimeRange;
072import org.apache.hadoop.hbase.util.Bytes;
073import org.apache.yetus.audience.InterfaceAudience;
074
075import org.apache.hbase.thirdparty.org.apache.commons.collections4.MapUtils;
076
077@InterfaceAudience.Private
078public class ThriftUtilities {
079
080  private ThriftUtilities() {
081    throw new UnsupportedOperationException("Can't initialize class");
082  }
083
084  /**
085   * Creates a {@link Get} (HBase) from a {@link TGet} (Thrift).
086   *
087   * This ignores any timestamps set on {@link TColumn} objects.
088   *
089   * @param in the <code>TGet</code> to convert
090   *
091   * @return <code>Get</code> object
092   *
093   * @throws IOException if an invalid time range or max version parameter is given
094   */
095  public static Get getFromThrift(TGet in) throws IOException {
096    Get out = new Get(in.getRow());
097
098    // Timestamp overwrites time range if both are set
099    if (in.isSetTimestamp()) {
100      out.setTimestamp(in.getTimestamp());
101    } else if (in.isSetTimeRange()) {
102      out.setTimeRange(in.getTimeRange().getMinStamp(), in.getTimeRange().getMaxStamp());
103    }
104
105    if (in.isSetMaxVersions()) {
106      out.setMaxVersions(in.getMaxVersions());
107    }
108
109    if (in.isSetFilterString()) {
110      ParseFilter parseFilter = new ParseFilter();
111      out.setFilter(parseFilter.parseFilterString(in.getFilterString()));
112    }
113
114    if (in.isSetAttributes()) {
115      addAttributes(out,in.getAttributes());
116    }
117
118    if (in.isSetAuthorizations()) {
119      out.setAuthorizations(new Authorizations(in.getAuthorizations().getLabels()));
120    }
121
122    if (!in.isSetColumns()) {
123      return out;
124    }
125
126    for (TColumn column : in.getColumns()) {
127      if (column.isSetQualifier()) {
128        out.addColumn(column.getFamily(), column.getQualifier());
129      } else {
130        out.addFamily(column.getFamily());
131      }
132    }
133
134    return out;
135  }
136
137  /**
138   * Converts multiple {@link TGet}s (Thrift) into a list of {@link Get}s (HBase).
139   *
140   * @param in list of <code>TGet</code>s to convert
141   *
142   * @return list of <code>Get</code> objects
143   *
144   * @throws IOException if an invalid time range or max version parameter is given
145   * @see #getFromThrift(TGet)
146   */
147  public static List<Get> getsFromThrift(List<TGet> in) throws IOException {
148    List<Get> out = new ArrayList<>(in.size());
149    for (TGet get : in) {
150      out.add(getFromThrift(get));
151    }
152    return out;
153  }
154
155  /**
156   * Creates a {@link TResult} (Thrift) from a {@link Result} (HBase).
157   *
158   * @param in the <code>Result</code> to convert
159   *
160   * @return converted result, returns an empty result if the input is <code>null</code>
161   */
162  public static TResult resultFromHBase(Result in) {
163    Cell[] raw = in.rawCells();
164    TResult out = new TResult();
165    byte[] row = in.getRow();
166    if (row != null) {
167      out.setRow(in.getRow());
168    }
169    List<TColumnValue> columnValues = new ArrayList<>(raw.length);
170    for (Cell kv : raw) {
171      TColumnValue col = new TColumnValue();
172      col.setFamily(CellUtil.cloneFamily(kv));
173      col.setQualifier(CellUtil.cloneQualifier(kv));
174      col.setTimestamp(kv.getTimestamp());
175      col.setValue(CellUtil.cloneValue(kv));
176      if (kv.getTagsLength() > 0) {
177        col.setTags(PrivateCellUtil.cloneTags(kv));
178      }
179      columnValues.add(col);
180    }
181    out.setColumnValues(columnValues);
182    return out;
183  }
184
185  /**
186   * Converts multiple {@link Result}s (HBase) into a list of {@link TResult}s (Thrift).
187   *
188   * @param in array of <code>Result</code>s to convert
189   *
190   * @return list of converted <code>TResult</code>s
191   *
192   * @see #resultFromHBase(Result)
193   */
194  public static List<TResult> resultsFromHBase(Result[] in) {
195    List<TResult> out = new ArrayList<>(in.length);
196    for (Result result : in) {
197      out.add(resultFromHBase(result));
198    }
199    return out;
200  }
201
202  /**
203   * Creates a {@link Put} (HBase) from a {@link TPut} (Thrift)
204   *
205   * @param in the <code>TPut</code> to convert
206   *
207   * @return converted <code>Put</code>
208   */
209  public static Put putFromThrift(TPut in) {
210    Put out;
211
212    if (in.isSetTimestamp()) {
213      out = new Put(in.getRow(), in.getTimestamp());
214    } else {
215      out = new Put(in.getRow());
216    }
217
218    if (in.isSetDurability()) {
219      out.setDurability(durabilityFromThrift(in.getDurability()));
220    }
221
222    for (TColumnValue columnValue : in.getColumnValues()) {
223      try {
224        if (columnValue.isSetTimestamp()) {
225          out.add(CellBuilderFactory.create(CellBuilderType.DEEP_COPY)
226              .setRow(out.getRow())
227              .setFamily(columnValue.getFamily())
228              .setQualifier(columnValue.getQualifier())
229              .setTimestamp(columnValue.getTimestamp())
230              .setType(Cell.Type.Put)
231              .setValue(columnValue.getValue())
232              .build());
233        } else {
234          out.add(CellBuilderFactory.create(CellBuilderType.DEEP_COPY)
235              .setRow(out.getRow())
236              .setFamily(columnValue.getFamily())
237              .setQualifier(columnValue.getQualifier())
238              .setTimestamp(out.getTimestamp())
239              .setType(Cell.Type.Put)
240              .setValue(columnValue.getValue())
241              .build());
242        }
243      } catch (IOException e) {
244        throw new IllegalArgumentException((e));
245      }
246    }
247
248    if (in.isSetAttributes()) {
249      addAttributes(out,in.getAttributes());
250    }
251
252    if (in.getCellVisibility() != null) {
253      out.setCellVisibility(new CellVisibility(in.getCellVisibility().getExpression()));
254    }
255
256    return out;
257  }
258
259  /**
260   * Converts multiple {@link TPut}s (Thrift) into a list of {@link Put}s (HBase).
261   *
262   * @param in list of <code>TPut</code>s to convert
263   *
264   * @return list of converted <code>Put</code>s
265   *
266   * @see #putFromThrift(TPut)
267   */
268  public static List<Put> putsFromThrift(List<TPut> in) {
269    List<Put> out = new ArrayList<>(in.size());
270    for (TPut put : in) {
271      out.add(putFromThrift(put));
272    }
273    return out;
274  }
275
276  /**
277   * Creates a {@link Delete} (HBase) from a {@link TDelete} (Thrift).
278   *
279   * @param in the <code>TDelete</code> to convert
280   *
281   * @return converted <code>Delete</code>
282   */
283  public static Delete deleteFromThrift(TDelete in) {
284    Delete out;
285
286    if (in.isSetColumns()) {
287      out = new Delete(in.getRow());
288      for (TColumn column : in.getColumns()) {
289        if (in.isSetDeleteType()) {
290          switch (in.getDeleteType()) {
291          case DELETE_COLUMN:
292            if (column.isSetTimestamp()) {
293              out.addColumn(column.getFamily(), column.getQualifier(), column.getTimestamp());
294            } else {
295              out.addColumn(column.getFamily(), column.getQualifier());
296            }
297            break;
298          case DELETE_COLUMNS:
299            if (column.isSetTimestamp()) {
300              out.addColumns(column.getFamily(), column.getQualifier(), column.getTimestamp());
301            } else {
302              out.addColumns(column.getFamily(), column.getQualifier());
303            }
304            break;
305          case DELETE_FAMILY:
306            if (column.isSetTimestamp()) {
307              out.addFamily(column.getFamily(), column.getTimestamp());
308            } else {
309              out.addFamily(column.getFamily());
310            }
311            break;
312          case DELETE_FAMILY_VERSION:
313            if (column.isSetTimestamp()) {
314              out.addFamilyVersion(column.getFamily(), column.getTimestamp());
315            } else {
316              throw new IllegalArgumentException(
317                  "Timestamp is required for TDelete with DeleteFamilyVersion type");
318            }
319            break;
320          }
321        } else {
322          throw new IllegalArgumentException("DeleteType is required for TDelete");
323        }
324      }
325    } else {
326      if (in.isSetTimestamp()) {
327        out = new Delete(in.getRow(), in.getTimestamp());
328      } else {
329        out = new Delete(in.getRow());
330      }
331    }
332
333    if (in.isSetAttributes()) {
334      addAttributes(out,in.getAttributes());
335    }
336
337    if (in.isSetDurability()) {
338      out.setDurability(durabilityFromThrift(in.getDurability()));
339    }
340
341    return out;
342  }
343
344  /**
345   * Converts multiple {@link TDelete}s (Thrift) into a list of {@link Delete}s (HBase).
346   *
347   * @param in list of <code>TDelete</code>s to convert
348   *
349   * @return list of converted <code>Delete</code>s
350   *
351   * @see #deleteFromThrift(TDelete)
352   */
353
354  public static List<Delete> deletesFromThrift(List<TDelete> in) {
355    List<Delete> out = new ArrayList<>(in.size());
356    for (TDelete delete : in) {
357      out.add(deleteFromThrift(delete));
358    }
359    return out;
360  }
361
362  public static TDelete deleteFromHBase(Delete in) {
363    TDelete out = new TDelete(ByteBuffer.wrap(in.getRow()));
364
365    List<TColumn> columns = new ArrayList<>(in.getFamilyCellMap().entrySet().size());
366    long rowTimestamp = in.getTimestamp();
367    if (rowTimestamp != HConstants.LATEST_TIMESTAMP) {
368      out.setTimestamp(rowTimestamp);
369    }
370
371    // Map<family, List<KeyValue>>
372    for (Map.Entry<byte[], List<org.apache.hadoop.hbase.Cell>> familyEntry:
373        in.getFamilyCellMap().entrySet()) {
374      TColumn column = new TColumn(ByteBuffer.wrap(familyEntry.getKey()));
375      for (org.apache.hadoop.hbase.Cell cell: familyEntry.getValue()) {
376        byte[] family = CellUtil.cloneFamily(cell);
377        byte[] qualifier = CellUtil.cloneQualifier(cell);
378        long timestamp = cell.getTimestamp();
379        if (family != null) {
380          column.setFamily(family);
381        }
382        if (qualifier != null) {
383          column.setQualifier(qualifier);
384        }
385        if (timestamp != HConstants.LATEST_TIMESTAMP) {
386          column.setTimestamp(timestamp);
387        }
388      }
389      columns.add(column);
390    }
391    out.setColumns(columns);
392
393    return out;
394  }
395
396  /**
397   * Creates a {@link RowMutations} (HBase) from a {@link TRowMutations} (Thrift)
398   *
399   * @param in the <code>TRowMutations</code> to convert
400   *
401   * @return converted <code>RowMutations</code>
402   */
403  public static RowMutations rowMutationsFromThrift(TRowMutations in) throws IOException {
404    List<TMutation> mutations = in.getMutations();
405    RowMutations out = new RowMutations(in.getRow(), mutations.size());
406    for (TMutation mutation : mutations) {
407      if (mutation.isSetPut()) {
408        out.add(putFromThrift(mutation.getPut()));
409      }
410      if (mutation.isSetDeleteSingle()) {
411        out.add(deleteFromThrift(mutation.getDeleteSingle()));
412      }
413    }
414    return out;
415  }
416
417  public static Scan scanFromThrift(TScan in) throws IOException {
418    Scan out = new Scan();
419
420    if (in.isSetStartRow())
421      out.setStartRow(in.getStartRow());
422    if (in.isSetStopRow())
423      out.setStopRow(in.getStopRow());
424    if (in.isSetCaching())
425      out.setCaching(in.getCaching());
426    if (in.isSetMaxVersions()) {
427      out.setMaxVersions(in.getMaxVersions());
428    }
429
430    if (in.isSetColumns()) {
431      for (TColumn column : in.getColumns()) {
432        if (column.isSetQualifier()) {
433          out.addColumn(column.getFamily(), column.getQualifier());
434        } else {
435          out.addFamily(column.getFamily());
436        }
437      }
438    }
439
440    TTimeRange timeRange = in.getTimeRange();
441    if (timeRange != null &&
442        timeRange.isSetMinStamp() && timeRange.isSetMaxStamp()) {
443      out.setTimeRange(timeRange.getMinStamp(), timeRange.getMaxStamp());
444    }
445
446    if (in.isSetBatchSize()) {
447      out.setBatch(in.getBatchSize());
448    }
449
450    if (in.isSetFilterString()) {
451      ParseFilter parseFilter = new ParseFilter();
452      out.setFilter(parseFilter.parseFilterString(in.getFilterString()));
453    }
454
455    if (in.isSetAttributes()) {
456      addAttributes(out,in.getAttributes());
457    }
458
459    if (in.isSetAuthorizations()) {
460      out.setAuthorizations(new Authorizations(in.getAuthorizations().getLabels()));
461    }
462
463    if (in.isSetReversed()) {
464      out.setReversed(in.isReversed());
465    }
466
467    if (in.isSetCacheBlocks()) {
468      out.setCacheBlocks(in.isCacheBlocks());
469    }
470
471    if (in.isSetColFamTimeRangeMap()) {
472      Map<ByteBuffer, TTimeRange> colFamTimeRangeMap = in.getColFamTimeRangeMap();
473      if (MapUtils.isNotEmpty(colFamTimeRangeMap)) {
474        for (Map.Entry<ByteBuffer, TTimeRange> entry : colFamTimeRangeMap.entrySet()) {
475          out.setColumnFamilyTimeRange(Bytes.toBytes(entry.getKey()),
476              entry.getValue().getMinStamp(), entry.getValue().getMaxStamp());
477        }
478      }
479    }
480
481    if (in.isSetReadType()) {
482      out.setReadType(readTypeFromThrift(in.getReadType()));
483    }
484
485    if (in.isSetLimit()) {
486      out.setLimit(in.getLimit());
487    }
488
489    return out;
490  }
491
492  public static Increment incrementFromThrift(TIncrement in) throws IOException {
493    Increment out = new Increment(in.getRow());
494    for (TColumnIncrement column : in.getColumns()) {
495      out.addColumn(column.getFamily(), column.getQualifier(), column.getAmount());
496    }
497
498    if (in.isSetAttributes()) {
499      addAttributes(out,in.getAttributes());
500    }
501
502    if (in.isSetDurability()) {
503      out.setDurability(durabilityFromThrift(in.getDurability()));
504    }
505
506    if(in.getCellVisibility() != null) {
507      out.setCellVisibility(new CellVisibility(in.getCellVisibility().getExpression()));
508    }
509
510    return out;
511  }
512
513  public static Append appendFromThrift(TAppend append) throws IOException {
514    Append out = new Append(append.getRow());
515    for (TColumnValue column : append.getColumns()) {
516      out.addColumn(column.getFamily(), column.getQualifier(), column.getValue());
517    }
518
519    if (append.isSetAttributes()) {
520      addAttributes(out, append.getAttributes());
521    }
522
523    if (append.isSetDurability()) {
524      out.setDurability(durabilityFromThrift(append.getDurability()));
525    }
526
527    if(append.getCellVisibility() != null) {
528      out.setCellVisibility(new CellVisibility(append.getCellVisibility().getExpression()));
529    }
530
531    return out;
532  }
533
534  public static THRegionLocation regionLocationFromHBase(HRegionLocation hrl) {
535    HRegionInfo hri = hrl.getRegionInfo();
536    ServerName serverName = hrl.getServerName();
537
538    THRegionInfo thRegionInfo = new THRegionInfo();
539    THRegionLocation thRegionLocation = new THRegionLocation();
540    TServerName tServerName = new TServerName();
541
542    tServerName.setHostName(serverName.getHostname());
543    tServerName.setPort(serverName.getPort());
544    tServerName.setStartCode(serverName.getStartcode());
545
546    thRegionInfo.setTableName(hri.getTable().getName());
547    thRegionInfo.setEndKey(hri.getEndKey());
548    thRegionInfo.setStartKey(hri.getStartKey());
549    thRegionInfo.setOffline(hri.isOffline());
550    thRegionInfo.setSplit(hri.isSplit());
551    thRegionInfo.setReplicaId(hri.getReplicaId());
552
553    thRegionLocation.setRegionInfo(thRegionInfo);
554    thRegionLocation.setServerName(tServerName);
555
556    return thRegionLocation;
557  }
558
559  public static List<THRegionLocation> regionLocationsFromHBase(List<HRegionLocation> locations) {
560    List<THRegionLocation> tlocations = new ArrayList<>(locations.size());
561    for (HRegionLocation hrl:locations) {
562      tlocations.add(regionLocationFromHBase(hrl));
563    }
564    return tlocations;
565  }
566
567  /**
568   * Adds all the attributes into the Operation object
569   */
570  private static void addAttributes(OperationWithAttributes op,
571                                    Map<ByteBuffer, ByteBuffer> attributes) {
572    if (attributes == null || attributes.isEmpty()) {
573      return;
574    }
575    for (Map.Entry<ByteBuffer, ByteBuffer> entry : attributes.entrySet()) {
576      String name = Bytes.toStringBinary(getBytes(entry.getKey()));
577      byte[] value =  getBytes(entry.getValue());
578      op.setAttribute(name, value);
579    }
580  }
581
582  private static Durability durabilityFromThrift(TDurability tDurability) {
583    switch (tDurability.getValue()) {
584      case 1: return Durability.SKIP_WAL;
585      case 2: return Durability.ASYNC_WAL;
586      case 3: return Durability.SYNC_WAL;
587      case 4: return Durability.FSYNC_WAL;
588      default: return null;
589    }
590  }
591
592  public static CompareOperator compareOpFromThrift(TCompareOp tCompareOp) {
593    switch (tCompareOp.getValue()) {
594      case 0: return CompareOperator.LESS;
595      case 1: return CompareOperator.LESS_OR_EQUAL;
596      case 2: return CompareOperator.EQUAL;
597      case 3: return CompareOperator.NOT_EQUAL;
598      case 4: return CompareOperator.GREATER_OR_EQUAL;
599      case 5: return CompareOperator.GREATER;
600      case 6: return CompareOperator.NO_OP;
601      default: return null;
602    }
603  }
604
605  private static ReadType readTypeFromThrift(TReadType tReadType) {
606    switch (tReadType.getValue()) {
607      case 1: return ReadType.DEFAULT;
608      case 2: return ReadType.STREAM;
609      case 3: return ReadType.PREAD;
610      default: return null;
611    }
612  }
613}