View Javadoc

1   /*
2    *
3    * Licensed to the Apache Software Foundation (ASF) under one
4    * or more contributor license agreements.  See the NOTICE file
5    * distributed with this work for additional information
6    * regarding copyright ownership.  The ASF licenses this file
7    * to you under the Apache License, Version 2.0 (the
8    * "License"); you may not use this file except in compliance
9    * with the License.  You may obtain a copy of the License at
10   *
11   *     http://www.apache.org/licenses/LICENSE-2.0
12   *
13   * Unless required by applicable law or agreed to in writing, software
14   * distributed under the License is distributed on an "AS IS" BASIS,
15   * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
16   * See the License for the specific language governing permissions and
17   * limitations under the License.
18   */
19  
20  package org.apache.hadoop.hbase.filter;
21  
22  import java.util.ArrayList;
23  
24  import org.apache.hadoop.hbase.util.ByteStringer;
25  import org.apache.hadoop.hbase.classification.InterfaceAudience;
26  import org.apache.hadoop.hbase.classification.InterfaceStability;
27  import org.apache.hadoop.hbase.Cell;
28  import org.apache.hadoop.hbase.KeyValueUtil;
29  import org.apache.hadoop.hbase.exceptions.DeserializationException;
30  import org.apache.hadoop.hbase.protobuf.generated.FilterProtos;
31  import org.apache.hadoop.hbase.util.Bytes;
32  
33  import com.google.common.base.Preconditions;
34  import com.google.protobuf.InvalidProtocolBufferException;
35  
36  /**
37   * This filter is used for selecting only those keys with columns that matches
38   * a particular prefix. For example, if prefix is 'an', it will pass keys with
39   * columns like 'and', 'anti' but not keys with columns like 'ball', 'act'.
40   */
41  @InterfaceAudience.Public
42  @InterfaceStability.Stable
43  public class ColumnPrefixFilter extends FilterBase {
44    protected byte [] prefix = null;
45  
46    public ColumnPrefixFilter(final byte [] prefix) {
47      this.prefix = prefix;
48    }
49  
50    public byte[] getPrefix() {
51      return prefix;
52    }
53  
54    @Override
55    public ReturnCode filterKeyValue(Cell kv) {
56      if (this.prefix == null || kv.getQualifierArray() == null) {
57        return ReturnCode.INCLUDE;
58      } else {
59        return filterColumn(kv.getQualifierArray(), kv.getQualifierOffset(), kv.getQualifierLength());
60      }
61    }
62  
63    // Override here explicitly as the method in super class FilterBase might do a KeyValue recreate.
64    // See HBASE-12068
65    @Override
66    public Cell transformCell(Cell v) {
67      return v;
68    }
69  
70    public ReturnCode filterColumn(byte[] buffer, int qualifierOffset, int qualifierLength) {
71      if (qualifierLength < prefix.length) {
72        int cmp = Bytes.compareTo(buffer, qualifierOffset, qualifierLength, this.prefix, 0,
73            qualifierLength);
74        if (cmp <= 0) {
75          return ReturnCode.SEEK_NEXT_USING_HINT;
76        } else {
77          return ReturnCode.NEXT_ROW;
78        }
79      } else {
80        int cmp = Bytes.compareTo(buffer, qualifierOffset, this.prefix.length, this.prefix, 0,
81            this.prefix.length);
82        if (cmp < 0) {
83          return ReturnCode.SEEK_NEXT_USING_HINT;
84        } else if (cmp > 0) {
85          return ReturnCode.NEXT_ROW;
86        } else {
87          return ReturnCode.INCLUDE;
88        }
89      }
90    }
91  
92    public static Filter createFilterFromArguments(ArrayList<byte []> filterArguments) {
93      Preconditions.checkArgument(filterArguments.size() == 1,
94                                  "Expected 1 but got: %s", filterArguments.size());
95      byte [] columnPrefix = ParseFilter.removeQuotesFromByteArray(filterArguments.get(0));
96      return new ColumnPrefixFilter(columnPrefix);
97    }
98  
99    /**
100    * @return The filter serialized using pb
101    */
102   public byte [] toByteArray() {
103     FilterProtos.ColumnPrefixFilter.Builder builder =
104       FilterProtos.ColumnPrefixFilter.newBuilder();
105     if (this.prefix != null) builder.setPrefix(ByteStringer.wrap(this.prefix));
106     return builder.build().toByteArray();
107   }
108 
109   /**
110    * @param pbBytes A pb serialized {@link ColumnPrefixFilter} instance
111    * @return An instance of {@link ColumnPrefixFilter} made from <code>bytes</code>
112    * @throws org.apache.hadoop.hbase.exceptions.DeserializationException
113    * @see #toByteArray
114    */
115   public static ColumnPrefixFilter parseFrom(final byte [] pbBytes)
116   throws DeserializationException {
117     FilterProtos.ColumnPrefixFilter proto;
118     try {
119       proto = FilterProtos.ColumnPrefixFilter.parseFrom(pbBytes);
120     } catch (InvalidProtocolBufferException e) {
121       throw new DeserializationException(e);
122     }
123     return new ColumnPrefixFilter(proto.getPrefix().toByteArray());
124   }
125 
126   /**
127    * @param other
128    * @return true if and only if the fields of the filter that are serialized
129    * are equal to the corresponding fields in other.  Used for testing.
130    */
131   boolean areSerializedFieldsEqual(Filter o) {
132    if (o == this) return true;
133    if (!(o instanceof ColumnPrefixFilter)) return false;
134 
135    ColumnPrefixFilter other = (ColumnPrefixFilter)o;
136     return Bytes.equals(this.getPrefix(), other.getPrefix());
137   }
138 
139   @Override
140   public Cell getNextCellHint(Cell kv) {
141     return KeyValueUtil.createFirstOnRow(
142         kv.getRowArray(), kv.getRowOffset(), kv.getRowLength(), kv.getFamilyArray(),
143         kv.getFamilyOffset(), kv.getFamilyLength(), prefix, 0, prefix.length);
144   }
145 
146   @Override
147   public String toString() {
148     return this.getClass().getSimpleName() + " " + Bytes.toStringBinary(this.prefix);
149   }
150 }