View Javadoc

1   /**
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
9    *
10   *     http://www.apache.org/licenses/LICENSE-2.0
11   *
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.
17   */
18  package org.apache.hadoop.hbase.types;
19  
20  import java.util.Iterator;
21  
22  import org.apache.hadoop.hbase.classification.InterfaceAudience;
23  import org.apache.hadoop.hbase.classification.InterfaceStability;
24  import org.apache.hadoop.hbase.util.Order;
25  import org.apache.hadoop.hbase.util.PositionedByteRange;
26  
27  /**
28   * <p>
29   * {@code Struct} is a simple {@link DataType} for implementing "compound
30   * rowkey" and "compound qualifier" schema design strategies.
31   * </p>
32   * <h3>Encoding</h3>
33   * <p>
34   * {@code Struct} member values are encoded onto the target byte[] in the order
35   * in which they are declared. A {@code Struct} may be used as a member of
36   * another {@code Struct}. {@code Struct}s are not {@code nullable} but their
37   * component fields may be.
38   * </p>
39   * <h3>Trailing Nulls</h3>
40   * <p>
41   * {@code Struct} treats the right-most nullable field members as special.
42   * Rather than writing null values to the output buffer, {@code Struct} omits
43   * those records all together. When reading back a value, it will look for the
44   * scenario where the end of the buffer has been reached but there are still
45   * nullable fields remaining in the {@code Struct} definition. When this
46   * happens, it will produce null entries for the remaining values. For example:
47   * </p>
48   * <pre>
49   * StructBuilder builder = new StructBuilder()
50   *     .add(OrderedNumeric.ASCENDING) // nullable
51   *     .add(OrderedString.ASCENDING)  // nullable
52   * Struct shorter = builder.toStruct();
53   * Struct longer = builder.add(OrderedNumeric.ASCENDING) // nullable
54   *     .toStruct();
55   *
56   * PositionedByteRange buf1 = new SimplePositionedByteRange(7);
57   * PositionedByteRange buf2 = new SimplePositionedByteRange(7);
58   * Object[] val = new Object[] { BigDecimal.ONE, "foo" };
59   * shorter.encode(buf1, val); // write short value with short Struct
60   * buf1.setPosition(0); // reset position marker, prepare for read
61   * longer.decode(buf1); // =&gt; { BigDecimal.ONE, "foo", null } ; long Struct reads implied null
62   * longer.encode(buf2, val); // write short value with long struct
63   * Bytes.equals(buf1.getBytes(), buf2.getBytes()); // =&gt; true; long Struct skips writing null
64   * </pre>
65   * <h3>Sort Order</h3>
66   * <p>
67   * {@code Struct} instances sort according to the composite order of their
68   * fields, that is, left-to-right and depth-first. This can also be thought of
69   * as lexicographic comparison of concatenated members.
70   * </p>
71   * <p>
72   * {@link StructIterator} is provided as a convenience for consuming the
73   * sequence of values. Users may find it more appropriate to provide their own
74   * custom {@link DataType} for encoding application objects rather than using
75   * this {@code Object[]} implementation. Examples are provided in test.
76   * </p>
77   * @see StructIterator
78   * @see DataType#isNullable()
79   */
80  @InterfaceAudience.Public
81  @InterfaceStability.Evolving
82  public class Struct implements DataType<Object[]> {
83  
84    @SuppressWarnings("rawtypes")
85    protected final DataType[] fields;
86    protected final boolean isOrderPreserving;
87    protected final boolean isSkippable;
88  
89    /**
90     * Create a new {@code Struct} instance defined as the sequence of
91     * {@code HDataType}s in {@code memberTypes}.
92     * <p>
93     * A {@code Struct} is {@code orderPreserving} when all of its fields
94     * are {@code orderPreserving}. A {@code Struct} is {@code skippable} when
95     * all of its fields are {@code skippable}.
96     * </p>
97     */
98    @SuppressWarnings("rawtypes")
99    public Struct(DataType[] memberTypes) {
100     this.fields = memberTypes;
101     // a Struct is not orderPreserving when any of its fields are not.
102     boolean preservesOrder = true;
103     // a Struct is not skippable when any of its fields are not.
104     boolean skippable = true;
105     for (int i = 0; i < this.fields.length; i++) {
106       DataType dt = this.fields[i];
107       if (!dt.isOrderPreserving()) preservesOrder = false;
108       if (i < this.fields.length - 2 && !dt.isSkippable()) {
109         throw new IllegalArgumentException("Field in position " + i
110           + " is not skippable. Non-right-most struct fields must be skippable.");
111       }
112       if (!dt.isSkippable()) skippable = false;
113     }
114     this.isOrderPreserving = preservesOrder;
115     this.isSkippable = skippable;
116   }
117 
118   @Override
119   public boolean isOrderPreserving() { return isOrderPreserving; }
120 
121   @Override
122   public Order getOrder() { return null; }
123 
124   @Override
125   public boolean isNullable() { return false; }
126 
127   @Override
128   public boolean isSkippable() { return isSkippable; }
129 
130   @SuppressWarnings("unchecked")
131   @Override
132   public int encodedLength(Object[] val) {
133     assert fields.length >= val.length;
134     int sum = 0;
135     for (int i = 0; i < val.length; i++)
136       sum += fields[i].encodedLength(val[i]);
137     return sum;
138   }
139 
140   @Override
141   public Class<Object[]> encodedClass() { return Object[].class; }
142 
143   /**
144    * Retrieve an {@link Iterator} over the values encoded in {@code src}.
145    * {@code src}'s position is consumed by consuming this iterator.
146    */
147   public StructIterator iterator(PositionedByteRange src) {
148     return new StructIterator(src, fields);
149   }
150 
151   @Override
152   public int skip(PositionedByteRange src) {
153     StructIterator it = iterator(src);
154     int skipped = 0;
155     while (it.hasNext())
156       skipped += it.skip();
157     return skipped;
158   }
159 
160   @Override
161   public Object[] decode(PositionedByteRange src) {
162     int i = 0;
163     Object[] ret = new Object[fields.length];
164     Iterator<Object> it = iterator(src);
165     while (it.hasNext())
166       ret[i++] = it.next();
167     return ret;
168   }
169 
170   /**
171    * Read the field at {@code index}. {@code src}'s position is not affected.
172    */
173   public Object decode(PositionedByteRange src, int index) {
174     assert index >= 0;
175     StructIterator it = iterator(src.shallowCopy());
176     for (; index > 0; index--)
177       it.skip();
178     return it.next();
179   }
180 
181   @SuppressWarnings("unchecked")
182   @Override
183   public int encode(PositionedByteRange dst, Object[] val) {
184     if (val.length == 0) return 0;
185     assert fields.length >= val.length;
186     int end, written = 0;
187     // find the last occurrence of a non-null or null and non-nullable value
188     for (end = val.length - 1; end > -1; end--) {
189       if (null != val[end] || (null == val[end] && !fields[end].isNullable())) break;
190     }
191     for (int i = 0; i <= end; i++) {
192       written += fields[i].encode(dst, val[i]);
193     }
194     return written;
195   }
196 }