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.filter;
019
020import java.io.IOException;
021import java.util.ArrayList;
022import java.util.Objects;
023import org.apache.hadoop.hbase.Cell;
024import org.apache.hadoop.hbase.CellUtil;
025import org.apache.hadoop.hbase.CompareOperator;
026import org.apache.hadoop.hbase.PrivateCellUtil;
027import org.apache.hadoop.hbase.exceptions.DeserializationException;
028import org.apache.hadoop.hbase.util.Bytes;
029import org.apache.yetus.audience.InterfaceAudience;
030
031import org.apache.hbase.thirdparty.com.google.common.base.Preconditions;
032import org.apache.hbase.thirdparty.com.google.protobuf.InvalidProtocolBufferException;
033import org.apache.hbase.thirdparty.com.google.protobuf.UnsafeByteOperations;
034
035import org.apache.hadoop.hbase.shaded.protobuf.ProtobufUtil;
036import org.apache.hadoop.hbase.shaded.protobuf.generated.FilterProtos;
037import org.apache.hadoop.hbase.shaded.protobuf.generated.HBaseProtos;
038
039/**
040 * Different from {@link SingleColumnValueFilter} which returns an <b>entire</b> row when specified
041 * condition is matched, {@link ColumnValueFilter} return the matched cell only.
042 * <p>
043 * This filter is used to filter cells based on column and value. It takes a
044 * {@link org.apache.hadoop.hbase.CompareOperator} operator (<, <=, =, !=, >, >=), and and a
045 * {@link ByteArrayComparable} comparator.
046 */
047@InterfaceAudience.Public
048public class ColumnValueFilter extends FilterBase {
049  private final byte[] family;
050  private final byte[] qualifier;
051  private final CompareOperator op;
052  private final ByteArrayComparable comparator;
053
054  // This flag is used to speed up seeking cells when matched column is found, such that following
055  // columns in the same row can be skipped faster by NEXT_ROW instead of NEXT_COL.
056  private boolean columnFound = false;
057
058  public ColumnValueFilter(final byte[] family, final byte[] qualifier, final CompareOperator op,
059    final byte[] value) {
060    this(family, qualifier, op, new BinaryComparator(value));
061  }
062
063  public ColumnValueFilter(final byte[] family, final byte[] qualifier, final CompareOperator op,
064    final ByteArrayComparable comparator) {
065    this.family = Preconditions.checkNotNull(family, "family should not be null.");
066    this.qualifier = qualifier == null ? new byte[0] : qualifier;
067    this.op = Preconditions.checkNotNull(op, "CompareOperator should not be null");
068    this.comparator = Preconditions.checkNotNull(comparator, "Comparator should not be null");
069  }
070
071  /** Returns operator */
072  public CompareOperator getCompareOperator() {
073    return op;
074  }
075
076  /** Returns the comparator */
077  public ByteArrayComparable getComparator() {
078    return comparator;
079  }
080
081  /** Returns the column family */
082  public byte[] getFamily() {
083    return family;
084  }
085
086  /** Returns the qualifier */
087  public byte[] getQualifier() {
088    return qualifier;
089  }
090
091  @Override
092  public void reset() throws IOException {
093    columnFound = false;
094  }
095
096  @Override
097  public boolean filterRowKey(Cell cell) throws IOException {
098    return false;
099  }
100
101  @Override
102  public ReturnCode filterCell(Cell c) throws IOException {
103    // 1. Check column match
104    if (!CellUtil.matchingColumn(c, this.family, this.qualifier)) {
105      return columnFound ? ReturnCode.NEXT_ROW : ReturnCode.NEXT_COL;
106    }
107    // Column found
108    columnFound = true;
109    // 2. Check value match:
110    // True means filter out, just skip this cell, else include it.
111    return compareValue(getCompareOperator(), getComparator(), c)
112      ? ReturnCode.SKIP
113      : ReturnCode.INCLUDE;
114  }
115
116  /**
117   * This method is used to determine a cell should be included or filtered out.
118   * @param op         one of operators {@link CompareOperator}
119   * @param comparator comparator used to compare cells.
120   * @param cell       cell to be compared.
121   * @return true means cell should be filtered out, included otherwise.
122   */
123  private boolean compareValue(final CompareOperator op, final ByteArrayComparable comparator,
124    final Cell cell) {
125    if (op == CompareOperator.NO_OP) {
126      return true;
127    }
128    int compareResult = PrivateCellUtil.compareValue(cell, comparator);
129    return CompareFilter.compare(op, compareResult);
130  }
131
132  /**
133   * Creating this filter by reflection, it is used by {@link ParseFilter},
134   * @param filterArguments arguments for creating a ColumnValueFilter
135   * @return a ColumnValueFilter
136   */
137  public static Filter createFilterFromArguments(ArrayList<byte[]> filterArguments) {
138    Preconditions.checkArgument(filterArguments.size() == 4, "Expect 4 arguments: %s",
139      filterArguments.size());
140    byte[] family = ParseFilter.removeQuotesFromByteArray(filterArguments.get(0));
141    byte[] qualifier = ParseFilter.removeQuotesFromByteArray(filterArguments.get(1));
142    CompareOperator operator = ParseFilter.createCompareOperator(filterArguments.get(2));
143    ByteArrayComparable comparator =
144      ParseFilter.createComparator(ParseFilter.removeQuotesFromByteArray(filterArguments.get(3)));
145
146    if (comparator instanceof RegexStringComparator || comparator instanceof SubstringComparator) {
147      if (operator != CompareOperator.EQUAL && operator != CompareOperator.NOT_EQUAL) {
148        throw new IllegalArgumentException("A regexstring comparator and substring comparator "
149          + "can only be used with EQUAL and NOT_EQUAL");
150      }
151    }
152
153    return new ColumnValueFilter(family, qualifier, operator, comparator);
154  }
155
156  /** Returns A pb instance to represent this instance. */
157  FilterProtos.ColumnValueFilter convert() {
158    FilterProtos.ColumnValueFilter.Builder builder = FilterProtos.ColumnValueFilter.newBuilder();
159
160    builder.setFamily(UnsafeByteOperations.unsafeWrap(this.family));
161    builder.setQualifier(UnsafeByteOperations.unsafeWrap(this.qualifier));
162    builder.setCompareOp(HBaseProtos.CompareType.valueOf(this.op.name()));
163    builder.setComparator(ProtobufUtil.toComparator(this.comparator));
164
165    return builder.build();
166  }
167
168  /**
169   * Parse a serialized representation of {@link ColumnValueFilter}
170   * @param pbBytes A pb serialized {@link ColumnValueFilter} instance
171   * @return An instance of {@link ColumnValueFilter} made from <code>bytes</code>
172   * @throws DeserializationException if an error occurred
173   * @see #toByteArray
174   */
175  public static ColumnValueFilter parseFrom(final byte[] pbBytes) throws DeserializationException {
176    FilterProtos.ColumnValueFilter proto;
177    try {
178      proto = FilterProtos.ColumnValueFilter.parseFrom(pbBytes);
179    } catch (InvalidProtocolBufferException e) {
180      throw new DeserializationException(e);
181    }
182
183    final CompareOperator compareOp = CompareOperator.valueOf(proto.getCompareOp().name());
184    final ByteArrayComparable comparator;
185    try {
186      comparator = ProtobufUtil.toComparator(proto.getComparator());
187    } catch (IOException ioe) {
188      throw new DeserializationException(ioe);
189    }
190
191    return new ColumnValueFilter(proto.getFamily().toByteArray(),
192      proto.getQualifier().toByteArray(), compareOp, comparator);
193  }
194
195  @Override
196  public byte[] toByteArray() throws IOException {
197    return convert().toByteArray();
198  }
199
200  /**
201   * Returns true if and only if the fields of the filter that are serialized are equal to the
202   * corresponding fields in other. Used for testing.
203   */
204  @Override
205  boolean areSerializedFieldsEqual(Filter o) {
206    if (o == this) {
207      return true;
208    } else if (!(o instanceof ColumnValueFilter)) {
209      return false;
210    }
211    ColumnValueFilter other = (ColumnValueFilter) o;
212    return Bytes.equals(this.getFamily(), other.getFamily())
213      && Bytes.equals(this.getQualifier(), other.getQualifier())
214      && this.getCompareOperator().equals(other.getCompareOperator())
215      && this.getComparator().areSerializedFieldsEqual(other.getComparator());
216  }
217
218  @Override
219  public boolean isFamilyEssential(byte[] name) throws IOException {
220    return Bytes.equals(name, this.family);
221  }
222
223  @Override
224  public String toString() {
225    return String.format("%s (%s, %s, %s, %s)", getClass().getSimpleName(),
226      Bytes.toStringBinary(this.family), Bytes.toStringBinary(this.qualifier), this.op.name(),
227      Bytes.toStringBinary(this.comparator.getValue()));
228  }
229
230  @Override
231  public boolean equals(Object obj) {
232    return obj instanceof Filter && areSerializedFieldsEqual((Filter) obj);
233  }
234
235  @Override
236  public int hashCode() {
237    return Objects.hash(Bytes.hashCode(getFamily()), Bytes.hashCode(getQualifier()),
238      getCompareOperator(), getComparator());
239  }
240}