001/*
002 * Licensed to the Apache Software Foundation (ASF) under one
003 * or more contributor license agreements.  See the NOTICE file
004 * distributed with this work for additional information
005 * regarding copyright ownership.  The ASF licenses this file
006 * to you under the Apache License, Version 2.0 (the
007 * "License"); you may not use this file except in compliance
008 * with the License.  You may obtain a copy of the License at
009 *
010 *     http://www.apache.org/licenses/LICENSE-2.0
011 *
012 * Unless required by applicable law or agreed to in writing, software
013 * distributed under the License is distributed on an "AS IS" BASIS,
014 * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
015 * See the License for the specific language governing permissions and
016 * limitations under the License.
017 */
018package org.apache.hadoop.hbase;
019
020import java.util.Comparator;
021import org.apache.hadoop.hbase.KeyValue.Type;
022import org.apache.hadoop.hbase.util.ByteBufferUtils;
023import org.apache.hadoop.hbase.util.Bytes;
024import org.apache.yetus.audience.InterfaceAudience;
025import org.apache.yetus.audience.InterfaceStability;
026
027/**
028 * Compare two HBase cells.  Do not use this method comparing <code>-ROOT-</code> or
029 * <code>hbase:meta</code> cells.  Cells from these tables need a specialized comparator, one that
030 * takes account of the special formatting of the row where we have commas to delimit table from
031 * regionname, from row.  See KeyValue for how it has a special comparator to do hbase:meta cells
032 * and yet another for -ROOT-.
033 * <p>While using this comparator for {{@link #compareRows(Cell, Cell)} et al, the hbase:meta cells
034 * format should be taken into consideration, for which the instance of this comparator
035 * should be used.  In all other cases the static APIs in this comparator would be enough
036 * <p>HOT methods. We spend a good portion of CPU comparing. Anything that makes the compare
037 * faster will likely manifest at the macro level. See also
038 * {@link BBKVComparator}. Use it when mostly {@link ByteBufferKeyValue}s.
039 * </p>
040 */
041@edu.umd.cs.findbugs.annotations.SuppressWarnings(
042    value="UNKNOWN",
043    justification="Findbugs doesn't like the way we are negating the result of a compare in below")
044@InterfaceAudience.Private
045@InterfaceStability.Evolving
046public class CellComparatorImpl implements CellComparator {
047
048  /**
049   * Comparator for plain key/values; i.e. non-catalog table key/values. Works on Key portion
050   * of KeyValue only.
051   */
052  public static final CellComparatorImpl COMPARATOR = new CellComparatorImpl();
053
054  @Override
055  public final int compare(final Cell a, final Cell b) {
056    return compare(a, b, false);
057  }
058
059  @Override
060  public int compare(final Cell a, final Cell b, boolean ignoreSequenceid) {
061
062    int diff = 0;
063    // "Peel off" the most common path.
064    if (a instanceof ByteBufferKeyValue && b instanceof ByteBufferKeyValue) {
065      diff = BBKVComparator.compare((ByteBufferKeyValue)a, (ByteBufferKeyValue)b, ignoreSequenceid);
066      if (diff != 0) {
067        return diff;
068      }
069    } else {
070      diff = compareRows(a, b);
071      if (diff != 0) {
072        return diff;
073      }
074
075      diff = compareWithoutRow(a, b);
076      if (diff != 0) {
077        return diff;
078      }
079    }
080
081    // Negate following comparisons so later edits show up first mvccVersion: later sorts first
082    return ignoreSequenceid? diff: Long.compare(b.getSequenceId(), a.getSequenceId());
083  }
084
085  /**
086   * Compares the family and qualifier part of the cell
087   * @return 0 if both cells are equal, 1 if left cell is bigger than right, -1 otherwise
088   */
089  public final int compareColumns(final Cell left, final Cell right) {
090    int diff = compareFamilies(left, right);
091    if (diff != 0) {
092      return diff;
093    }
094    return compareQualifiers(left, right);
095  }
096
097  /**
098   * Compare the families of left and right cell
099   * @return 0 if both cells are equal, 1 if left cell is bigger than right, -1 otherwise
100   */
101  @Override
102  public final int compareFamilies(Cell left, Cell right) {
103    if (left instanceof ByteBufferExtendedCell && right instanceof ByteBufferExtendedCell) {
104      return ByteBufferUtils.compareTo(((ByteBufferExtendedCell) left).getFamilyByteBuffer(),
105          ((ByteBufferExtendedCell) left).getFamilyPosition(), left.getFamilyLength(),
106          ((ByteBufferExtendedCell) right).getFamilyByteBuffer(),
107          ((ByteBufferExtendedCell) right).getFamilyPosition(), right.getFamilyLength());
108    }
109    if (left instanceof ByteBufferExtendedCell) {
110      return ByteBufferUtils.compareTo(((ByteBufferExtendedCell) left).getFamilyByteBuffer(),
111          ((ByteBufferExtendedCell) left).getFamilyPosition(), left.getFamilyLength(),
112          right.getFamilyArray(), right.getFamilyOffset(), right.getFamilyLength());
113    }
114    if (right instanceof ByteBufferExtendedCell) {
115      // Notice how we flip the order of the compare here. We used to negate the return value but
116      // see what FindBugs says
117      // http://findbugs.sourceforge.net/bugDescriptions.html#RV_NEGATING_RESULT_OF_COMPARETO
118      // It suggest flipping the order to get same effect and 'safer'.
119      return ByteBufferUtils.compareTo(
120          left.getFamilyArray(), left.getFamilyOffset(), left.getFamilyLength(),
121          ((ByteBufferExtendedCell)right).getFamilyByteBuffer(),
122          ((ByteBufferExtendedCell)right).getFamilyPosition(), right.getFamilyLength());
123    }
124    return Bytes.compareTo(left.getFamilyArray(), left.getFamilyOffset(), left.getFamilyLength(),
125        right.getFamilyArray(), right.getFamilyOffset(), right.getFamilyLength());
126  }
127
128  /**
129   * Compare the qualifiers part of the left and right cells.
130   * @return 0 if both cells are equal, 1 if left cell is bigger than right, -1 otherwise
131   */
132  @Override
133  public final int compareQualifiers(Cell left, Cell right) {
134    if (left instanceof ByteBufferExtendedCell && right instanceof ByteBufferExtendedCell) {
135      return ByteBufferUtils
136          .compareTo(((ByteBufferExtendedCell) left).getQualifierByteBuffer(),
137              ((ByteBufferExtendedCell) left).getQualifierPosition(),
138              left.getQualifierLength(), ((ByteBufferExtendedCell) right).getQualifierByteBuffer(),
139              ((ByteBufferExtendedCell) right).getQualifierPosition(),
140              right.getQualifierLength());
141    }
142    if (left instanceof ByteBufferExtendedCell) {
143      return ByteBufferUtils.compareTo(((ByteBufferExtendedCell) left).getQualifierByteBuffer(),
144          ((ByteBufferExtendedCell) left).getQualifierPosition(), left.getQualifierLength(),
145          right.getQualifierArray(), right.getQualifierOffset(), right.getQualifierLength());
146    }
147    if (right instanceof ByteBufferExtendedCell) {
148      // Notice how we flip the order of the compare here. We used to negate the return value but
149      // see what FindBugs says
150      // http://findbugs.sourceforge.net/bugDescriptions.html#RV_NEGATING_RESULT_OF_COMPARETO
151      // It suggest flipping the order to get same effect and 'safer'.
152      return ByteBufferUtils.compareTo(left.getQualifierArray(),
153          left.getQualifierOffset(), left.getQualifierLength(),
154          ((ByteBufferExtendedCell)right).getQualifierByteBuffer(),
155          ((ByteBufferExtendedCell)right).getQualifierPosition(), right.getQualifierLength());
156    }
157    return Bytes.compareTo(left.getQualifierArray(), left.getQualifierOffset(),
158        left.getQualifierLength(), right.getQualifierArray(), right.getQualifierOffset(),
159        right.getQualifierLength());
160  }
161
162  /**
163   * Compares the rows of the left and right cell.
164   * For the hbase:meta case this method is overridden such that it can handle hbase:meta cells.
165   * The caller should ensure using the appropriate comparator for hbase:meta.
166   * @return 0 if both cells are equal, 1 if left cell is bigger than right, -1 otherwise
167   */
168  @Override
169  public int compareRows(final Cell left, final Cell right) {
170    return compareRows(left, left.getRowLength(), right, right.getRowLength());
171  }
172
173  static int compareRows(final Cell left, int leftRowLength, final Cell right, int rightRowLength) {
174    // left and right can be exactly the same at the beginning of a row
175    if (left == right) {
176      return 0;
177    }
178    if (left instanceof ByteBufferExtendedCell && right instanceof ByteBufferExtendedCell) {
179      return ByteBufferUtils.compareTo(((ByteBufferExtendedCell) left).getRowByteBuffer(),
180          ((ByteBufferExtendedCell) left).getRowPosition(), leftRowLength,
181          ((ByteBufferExtendedCell) right).getRowByteBuffer(),
182          ((ByteBufferExtendedCell) right).getRowPosition(), rightRowLength);
183    }
184    if (left instanceof ByteBufferExtendedCell) {
185      return ByteBufferUtils.compareTo(((ByteBufferExtendedCell) left).getRowByteBuffer(),
186          ((ByteBufferExtendedCell) left).getRowPosition(), leftRowLength,
187          right.getRowArray(), right.getRowOffset(), rightRowLength);
188    }
189    if (right instanceof ByteBufferExtendedCell) {
190      // Notice how we flip the order of the compare here. We used to negate the return value but
191      // see what FindBugs says
192      // http://findbugs.sourceforge.net/bugDescriptions.html#RV_NEGATING_RESULT_OF_COMPARETO
193      // It suggest flipping the order to get same effect and 'safer'.
194      return ByteBufferUtils.compareTo(left.getRowArray(), left.getRowOffset(), leftRowLength,
195          ((ByteBufferExtendedCell)right).getRowByteBuffer(),
196          ((ByteBufferExtendedCell)right).getRowPosition(), rightRowLength);
197    }
198    return Bytes.compareTo(left.getRowArray(), left.getRowOffset(), left.getRowLength(),
199        right.getRowArray(), right.getRowOffset(), right.getRowLength());
200  }
201
202  /**
203   * Compares the row part of the cell with a simple plain byte[] like the
204   * stopRow in Scan. This should be used with context where for hbase:meta
205   * cells the {{@link MetaCellComparator#META_COMPARATOR} should be used
206   *
207   * @param left
208   *          the cell to be compared
209   * @param right
210   *          the kv serialized byte[] to be compared with
211   * @param roffset
212   *          the offset in the byte[]
213   * @param rlength
214   *          the length in the byte[]
215   * @return 0 if both cell and the byte[] are equal, 1 if the cell is bigger
216   *         than byte[], -1 otherwise
217   */
218  @Override
219  public int compareRows(Cell left, byte[] right, int roffset, int rlength) {
220    if (left instanceof ByteBufferExtendedCell) {
221      return ByteBufferUtils.compareTo(((ByteBufferExtendedCell) left).getRowByteBuffer(),
222          ((ByteBufferExtendedCell) left).getRowPosition(), left.getRowLength(), right,
223          roffset, rlength);
224    }
225    return Bytes.compareTo(left.getRowArray(), left.getRowOffset(), left.getRowLength(), right,
226        roffset, rlength);
227  }
228
229  @Override
230  public final int compareWithoutRow(final Cell left, final Cell right) {
231    // If the column is not specified, the "minimum" key type appears the
232    // latest in the sorted order, regardless of the timestamp. This is used
233    // for specifying the last key/value in a given row, because there is no
234    // "lexicographically last column" (it would be infinitely long). The
235    // "maximum" key type does not need this behavior.
236    // Copied from KeyValue. This is bad in that we can't do memcmp w/ special rules like this.
237    int lFamLength = left.getFamilyLength();
238    int rFamLength = right.getFamilyLength();
239    int lQualLength = left.getQualifierLength();
240    int rQualLength = right.getQualifierLength();
241    if (lFamLength + lQualLength == 0
242          && left.getTypeByte() == Type.Minimum.getCode()) {
243      // left is "bigger", i.e. it appears later in the sorted order
244      return 1;
245    }
246    if (rFamLength + rQualLength == 0
247        && right.getTypeByte() == Type.Minimum.getCode()) {
248      return -1;
249    }
250    if (lFamLength != rFamLength) {
251      // comparing column family is enough.
252      return compareFamilies(left, right);
253    }
254    // Compare cf:qualifier
255    int diff = compareColumns(left, right);
256    if (diff != 0) {
257      return diff;
258    }
259
260    diff = compareTimestamps(left.getTimestamp(), right.getTimestamp());
261    if (diff != 0) {
262      return diff;
263    }
264
265    // Compare types. Let the delete types sort ahead of puts; i.e. types
266    // of higher numbers sort before those of lesser numbers. Maximum (255)
267    // appears ahead of everything, and minimum (0) appears after
268    // everything.
269    return (0xff & right.getTypeByte()) - (0xff & left.getTypeByte());
270  }
271
272  @Override
273  public int compareTimestamps(final Cell left, final Cell right) {
274    return compareTimestamps(left.getTimestamp(), right.getTimestamp());
275  }
276
277  @Override
278  public int compareTimestamps(final long ltimestamp, final long rtimestamp) {
279    // Swap order we pass into compare so we get DESCENDING order.
280    return Long.compare(rtimestamp, ltimestamp);
281  }
282
283  @Override
284  public Comparator getSimpleComparator() {
285    return new BBKVComparator(this);
286  }
287
288  /**
289   * Utility method that makes a guess at comparator to use based off passed tableName.
290   * Use in extreme when no comparator specified.
291   * @return CellComparator to use going off the {@code tableName} passed.
292   */
293  public static CellComparator getCellComparator(TableName tableName) {
294    return getCellComparator(tableName.toBytes());
295  }
296
297  /**
298   * Utility method that makes a guess at comparator to use based off passed tableName.
299   * Use in extreme when no comparator specified.
300   * @return CellComparator to use going off the {@code tableName} passed.
301   */
302  public static CellComparator getCellComparator(byte [] tableName) {
303    // FYI, TableName.toBytes does not create an array; just returns existing array pointer.
304    return Bytes.equals(tableName, TableName.META_TABLE_NAME.toBytes())?
305      MetaCellComparator.META_COMPARATOR: CellComparatorImpl.COMPARATOR;
306  }
307}