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.constraint;
019
020import java.io.ByteArrayInputStream;
021import java.io.ByteArrayOutputStream;
022import java.io.DataOutputStream;
023import java.io.IOException;
024import java.lang.reflect.InvocationTargetException;
025import java.util.ArrayList;
026import java.util.Collections;
027import java.util.Comparator;
028import java.util.List;
029import java.util.Map;
030import java.util.Map.Entry;
031import java.util.regex.Pattern;
032import org.apache.hadoop.conf.Configuration;
033import org.apache.hadoop.hbase.HTableDescriptor;
034import org.apache.hadoop.hbase.client.TableDescriptor;
035import org.apache.hadoop.hbase.util.Bytes;
036import org.apache.hadoop.hbase.util.Pair;
037import org.apache.yetus.audience.InterfaceAudience;
038import org.slf4j.Logger;
039import org.slf4j.LoggerFactory;
040
041/**
042 * Utilities for adding/removing constraints from a table.
043 * <p>
044 * Constraints can be added on table load time, via the {@link HTableDescriptor}.
045 * <p>
046 * NOTE: this class is NOT thread safe. Concurrent setting/enabling/disabling of constraints can
047 * cause constraints to be run at incorrect times or not at all.
048 */
049@InterfaceAudience.Private
050public final class Constraints {
051  private static final int DEFAULT_PRIORITY = -1;
052
053  private Constraints() {
054  }
055
056  private static final Logger LOG = LoggerFactory.getLogger(Constraints.class);
057  private static final String CONSTRAINT_HTD_KEY_PREFIX = "constraint $";
058  private static final Pattern CONSTRAINT_HTD_ATTR_KEY_PATTERN =
059    Pattern.compile(CONSTRAINT_HTD_KEY_PREFIX, Pattern.LITERAL);
060
061  // configuration key for if the constraint is enabled
062  private static final String ENABLED_KEY = "_ENABLED";
063  // configuration key for the priority
064  private static final String PRIORITY_KEY = "_PRIORITY";
065
066  // smallest priority a constraiNt can have
067  private static final long MIN_PRIORITY = 0L;
068  // ensure a priority less than the smallest we could intentionally set
069  private static final long UNSET_PRIORITY = MIN_PRIORITY - 1;
070
071  private static String COUNTER_KEY = "hbase.constraint.counter";
072
073  /**
074   * Enable constraints on a table.
075   * <p>
076   * Currently, if you attempt to add a constraint to the table, then Constraints will automatically
077   * be turned on. table description to add the processor If the {@link ConstraintProcessor} CP
078   * couldn't be added to the table.
079   */
080  public static void enable(HTableDescriptor desc) throws IOException {
081    // if the CP has already been loaded, do nothing
082    String clazz = ConstraintProcessor.class.getName();
083    if (desc.hasCoprocessor(clazz)) {
084      return;
085    }
086
087    // add the constrain processor CP to the table
088    desc.addCoprocessor(clazz);
089  }
090
091  /**
092   * Turn off processing constraints for a given table, even if constraints have been turned on or
093   * added. {@link HTableDescriptor} where to disable {@link Constraint Constraints}.
094   */
095  public static void disable(HTableDescriptor desc) {
096    desc.removeCoprocessor(ConstraintProcessor.class.getName());
097  }
098
099  /**
100   * Remove all {@link Constraint Constraints} that have been added to the table and turn off the
101   * constraint processing.
102   * <p>
103   * All {@link Configuration Configurations} and their associated {@link Constraint} are removed.
104   * @param desc {@link HTableDescriptor} to remove {@link Constraint Constraints} from.
105   */
106  public static void remove(HTableDescriptor desc) {
107    // disable constraints
108    disable(desc);
109
110    // remove all the constraint settings
111    List<Bytes> keys = new ArrayList<>();
112    // loop through all the key, values looking for constraints
113    for (Map.Entry<Bytes, Bytes> e : desc.getValues().entrySet()) {
114      String key = Bytes.toString((e.getKey().get()));
115      String[] className = CONSTRAINT_HTD_ATTR_KEY_PATTERN.split(key);
116      if (className.length == 2) {
117        keys.add(e.getKey());
118      }
119    }
120    // now remove all the keys we found
121    for (Bytes key : keys) {
122      desc.remove(key);
123    }
124  }
125
126  /**
127   * Check to see if the Constraint is currently set. {@link HTableDescriptor} to check
128   * {@link Constraint} class to check for.
129   * @return <tt>true</tt> if the {@link Constraint} is present, even if it is disabled.
130   *         <tt>false</tt> otherwise.
131   */
132  public static boolean has(HTableDescriptor desc, Class<? extends Constraint> clazz) {
133    return getKeyValueForClass(desc, clazz) != null;
134  }
135
136  /**
137   * Get the kv {@link Entry} in the descriptor for the specified class
138   * @param desc  {@link HTableDescriptor} to read
139   * @param clazz To search for
140   * @return The {@link Pair} of {@literal <key, value>} in the table, if that class is present.
141   *         {@code NULL} otherwise.
142   */
143  private static Pair<String, String> getKeyValueForClass(HTableDescriptor desc,
144    Class<? extends Constraint> clazz) {
145    // get the serialized version of the constraint
146    String key = serializeConstraintClass(clazz);
147    String value = desc.getValue(key);
148
149    return value == null ? null : new Pair<>(key, value);
150  }
151
152  /**
153   * Add configuration-less constraints to the table.
154   * <p>
155   * This will overwrite any configuration associated with the previous constraint of the same
156   * class.
157   * <p>
158   * Each constraint, when added to the table, will have a specific priority, dictating the order in
159   * which the {@link Constraint} will be run. A {@link Constraint} earlier in the list will be run
160   * before those later in the list. The same logic applies between two Constraints over time
161   * (earlier added is run first on the regionserver). {@link HTableDescriptor} to add
162   * {@link Constraint Constraints} {@link Constraint Constraints} to add. All constraints are
163   * considered automatically enabled on add If constraint could not be serialized/added to table
164   */
165  public static void add(HTableDescriptor desc, Class<? extends Constraint>... constraints)
166    throws IOException {
167    // make sure constraints are enabled
168    enable(desc);
169    long priority = getNextPriority(desc);
170
171    // store each constraint
172    for (Class<? extends Constraint> clazz : constraints) {
173      addConstraint(desc, clazz, null, priority++);
174    }
175    updateLatestPriority(desc, priority);
176  }
177
178  /**
179   * Add constraints and their associated configurations to the table.
180   * <p>
181   * Adding the same constraint class twice will overwrite the first constraint's configuration
182   * <p>
183   * Each constraint, when added to the table, will have a specific priority, dictating the order in
184   * which the {@link Constraint} will be run. A {@link Constraint} earlier in the list will be run
185   * before those later in the list. The same logic applies between two Constraints over time
186   * (earlier added is run first on the regionserver). {@link HTableDescriptor} to add a
187   * {@link Constraint} {@link Pair} of a {@link Constraint} and its associated
188   * {@link Configuration}. The Constraint will be configured on load with the specified
189   * configuration.All constraints are considered automatically enabled on add if any constraint
190   * could not be deserialized. Assumes if 1 constraint is not loaded properly, something has gone
191   * terribly wrong and that all constraints need to be enforced.
192   */
193  public static void add(HTableDescriptor desc,
194    Pair<Class<? extends Constraint>, Configuration>... constraints) throws IOException {
195    enable(desc);
196    long priority = getNextPriority(desc);
197    for (Pair<Class<? extends Constraint>, Configuration> pair : constraints) {
198      addConstraint(desc, pair.getFirst(), pair.getSecond(), priority++);
199    }
200    updateLatestPriority(desc, priority);
201  }
202
203  /**
204   * Add a {@link Constraint} to the table with the given configuration
205   * <p>
206   * Each constraint, when added to the table, will have a specific priority, dictating the order in
207   * which the {@link Constraint} will be run. A {@link Constraint} added will run on the
208   * regionserver before those added to the {@link HTableDescriptor} later. table descriptor to the
209   * constraint to to be added configuration associated with the constraint if any constraint could
210   * not be deserialized. Assumes if 1 constraint is not loaded properly, something has gone
211   * terribly wrong and that all constraints need to be enforced.
212   */
213  public static void add(HTableDescriptor desc, Class<? extends Constraint> constraint,
214    Configuration conf) throws IOException {
215    enable(desc);
216    long priority = getNextPriority(desc);
217    addConstraint(desc, constraint, conf, priority++);
218
219    updateLatestPriority(desc, priority);
220  }
221
222  /**
223   * Write the raw constraint and configuration to the descriptor.
224   * <p>
225   * This method takes care of creating a new configuration based on the passed in configuration and
226   * then updating that with enabled and priority of the constraint.
227   * <p>
228   * When a constraint is added, it is automatically enabled.
229   */
230  private static void addConstraint(HTableDescriptor desc, Class<? extends Constraint> clazz,
231    Configuration conf, long priority) throws IOException {
232    writeConstraint(desc, serializeConstraintClass(clazz), configure(conf, true, priority));
233  }
234
235  /**
236   * Setup the configuration for a constraint as to whether it is enabled and its priority on which
237   * to base the new configuration <tt>true</tt> if it should be run relative to other constraints
238   * @return a new configuration, storable in the {@link HTableDescriptor}
239   */
240  private static Configuration configure(Configuration conf, boolean enabled, long priority) {
241    // create the configuration to actually be stored
242    // clone if possible, but otherwise just create an empty configuration
243    Configuration toWrite = conf == null ? new Configuration() : new Configuration(conf);
244
245    // update internal properties
246    toWrite.setBooleanIfUnset(ENABLED_KEY, enabled);
247
248    // set if unset long
249    if (toWrite.getLong(PRIORITY_KEY, UNSET_PRIORITY) == UNSET_PRIORITY) {
250      toWrite.setLong(PRIORITY_KEY, priority);
251    }
252
253    return toWrite;
254  }
255
256  /**
257   * Just write the class to a String representation of the class as a key for the
258   * {@link HTableDescriptor} Constraint class to convert to a {@link HTableDescriptor} key
259   * @return key to store in the {@link HTableDescriptor}
260   */
261  private static String serializeConstraintClass(Class<? extends Constraint> clazz) {
262    String constraintClazz = clazz.getName();
263    return CONSTRAINT_HTD_KEY_PREFIX + constraintClazz;
264  }
265
266  /**
267   * Write the given key and associated configuration to the {@link HTableDescriptor}
268   */
269  private static void writeConstraint(HTableDescriptor desc, String key, Configuration conf)
270    throws IOException {
271    // store the key and conf in the descriptor
272    desc.setValue(key, serializeConfiguration(conf));
273  }
274
275  /**
276   * Write the configuration to a String to write
277   * @return String representation of that configuration
278   */
279  private static String serializeConfiguration(Configuration conf) throws IOException {
280    // write the configuration out to the data stream
281    ByteArrayOutputStream bos = new ByteArrayOutputStream();
282    DataOutputStream dos = new DataOutputStream(bos);
283    conf.writeXml(dos);
284    dos.flush();
285    byte[] data = bos.toByteArray();
286    return Bytes.toString(data);
287  }
288
289  /**
290   * Read the {@link Configuration} stored in the byte stream. to read from
291   * @return A valid configuration
292   */
293  private static Configuration readConfiguration(byte[] bytes) throws IOException {
294    ByteArrayInputStream is = new ByteArrayInputStream(bytes);
295    Configuration conf = new Configuration(false);
296    conf.addResource(is);
297    return conf;
298  }
299
300  /**
301   * Read in the configuration from the String encoded configuration to read from
302   * @return A valid configuration if the configuration could not be read
303   */
304  private static Configuration readConfiguration(String bytes) throws IOException {
305    return readConfiguration(Bytes.toBytes(bytes));
306  }
307
308  private static long getNextPriority(HTableDescriptor desc) {
309    String value = desc.getValue(COUNTER_KEY);
310
311    long priority;
312    // get the current priority
313    if (value == null) {
314      priority = MIN_PRIORITY;
315    } else {
316      priority = Long.parseLong(value) + 1;
317    }
318
319    return priority;
320  }
321
322  private static void updateLatestPriority(HTableDescriptor desc, long priority) {
323    // update the max priority
324    desc.setValue(COUNTER_KEY, Long.toString(priority));
325  }
326
327  /**
328   * Update the configuration for the {@link Constraint}; does not change the order in which the
329   * constraint is run. {@link HTableDescriptor} to update {@link Constraint} to update to update
330   * the {@link Constraint} with. if the Constraint was not stored correctly if the Constraint was
331   * not present on this table.
332   */
333  public static void setConfiguration(HTableDescriptor desc, Class<? extends Constraint> clazz,
334    Configuration configuration) throws IOException, IllegalArgumentException {
335    // get the entry for this class
336    Pair<String, String> e = getKeyValueForClass(desc, clazz);
337
338    if (e == null) {
339      throw new IllegalArgumentException(
340        "Constraint: " + clazz.getName() + " is not associated with this table.");
341    }
342
343    // clone over the configuration elements
344    Configuration conf = new Configuration(configuration);
345
346    // read in the previous info about the constraint
347    Configuration internal = readConfiguration(e.getSecond());
348
349    // update the fields based on the previous settings
350    conf.setIfUnset(ENABLED_KEY, internal.get(ENABLED_KEY));
351    conf.setIfUnset(PRIORITY_KEY, internal.get(PRIORITY_KEY));
352
353    // update the current value
354    writeConstraint(desc, e.getFirst(), conf);
355  }
356
357  /**
358   * Remove the constraint (and associated information) for the table descriptor.
359   * {@link HTableDescriptor} to modify {@link Constraint} class to remove
360   */
361  public static void remove(HTableDescriptor desc, Class<? extends Constraint> clazz) {
362    String key = serializeConstraintClass(clazz);
363    desc.remove(key);
364  }
365
366  /**
367   * Enable the given {@link Constraint}. Retains all the information (e.g. Configuration) for the
368   * {@link Constraint}, but makes sure that it gets loaded on the table. {@link HTableDescriptor}
369   * to modify {@link Constraint} to enable If the constraint cannot be properly deserialized
370   */
371  public static void enableConstraint(HTableDescriptor desc, Class<? extends Constraint> clazz)
372    throws IOException {
373    changeConstraintEnabled(desc, clazz, true);
374  }
375
376  /**
377   * Disable the given {@link Constraint}. Retains all the information (e.g. Configuration) for the
378   * {@link Constraint}, but it just doesn't load the {@link Constraint} on the table.
379   * {@link HTableDescriptor} to modify {@link Constraint} to disable. if the constraint cannot be
380   * found
381   */
382  public static void disableConstraint(HTableDescriptor desc, Class<? extends Constraint> clazz)
383    throws IOException {
384    changeConstraintEnabled(desc, clazz, false);
385  }
386
387  /**
388   * Change the whether the constraint (if it is already present) is enabled or disabled.
389   */
390  private static void changeConstraintEnabled(HTableDescriptor desc,
391    Class<? extends Constraint> clazz, boolean enabled) throws IOException {
392    // get the original constraint
393    Pair<String, String> entry = getKeyValueForClass(desc, clazz);
394    if (entry == null) {
395      throw new IllegalArgumentException("Constraint: " + clazz.getName()
396        + " is not associated with this table. You can't enable it!");
397    }
398
399    // create a new configuration from that conf
400    Configuration conf = readConfiguration(entry.getSecond());
401
402    // set that it is enabled
403    conf.setBoolean(ENABLED_KEY, enabled);
404
405    // write it back out
406    writeConstraint(desc, entry.getFirst(), conf);
407  }
408
409  /**
410   * Check to see if the given constraint is enabled. {@link HTableDescriptor} to check.
411   * {@link Constraint} to check for
412   * @return <tt>true</tt> if the {@link Constraint} is present and enabled. <tt>false</tt>
413   *         otherwise. If the constraint has improperly stored in the table
414   */
415  public static boolean enabled(HTableDescriptor desc, Class<? extends Constraint> clazz)
416    throws IOException {
417    // get the kv
418    Pair<String, String> entry = getKeyValueForClass(desc, clazz);
419    // its not enabled so just return false. In fact, its not even present!
420    if (entry == null) {
421      return false;
422    }
423
424    // get the info about the constraint
425    Configuration conf = readConfiguration(entry.getSecond());
426
427    return conf.getBoolean(ENABLED_KEY, false);
428  }
429
430  /**
431   * Get the constraints stored in the table descriptor To read from To use when loading classes. If
432   * a special classloader is used on a region, for instance, then that should be the classloader
433   * used to load the constraints. This could also apply to unit-testing situation, where want to
434   * ensure that class is reloaded or not.
435   * @return List of configured {@link Constraint Constraints} if any part of reading/arguments
436   *         fails
437   */
438  static List<? extends Constraint> getConstraints(TableDescriptor desc, ClassLoader classloader)
439    throws IOException {
440    List<Constraint> constraints = new ArrayList<>();
441    // loop through all the key, values looking for constraints
442    for (Map.Entry<Bytes, Bytes> e : desc.getValues().entrySet()) {
443      // read out the constraint
444      String key = Bytes.toString(e.getKey().get()).trim();
445      String[] className = CONSTRAINT_HTD_ATTR_KEY_PATTERN.split(key);
446      if (className.length == 2) {
447        key = className[1];
448        if (LOG.isDebugEnabled()) {
449          LOG.debug("Loading constraint:" + key);
450        }
451
452        // read in the rest of the constraint
453        Configuration conf;
454        try {
455          conf = readConfiguration(e.getValue().get());
456        } catch (IOException e1) {
457          // long that we don't have a valid configuration stored, and move on.
458          LOG.warn("Corrupted configuration found for key:" + key + ",  skipping it.");
459          continue;
460        }
461        // if it is not enabled, skip it
462        if (!conf.getBoolean(ENABLED_KEY, false)) {
463          if (LOG.isDebugEnabled()) LOG.debug("Constraint: " + key + " is DISABLED - skipping it");
464          // go to the next constraint
465          continue;
466        }
467
468        try {
469          // add the constraint, now that we expect it to be valid.
470          Class<? extends Constraint> clazz =
471            classloader.loadClass(key).asSubclass(Constraint.class);
472          Constraint constraint = clazz.getDeclaredConstructor().newInstance();
473          constraint.setConf(conf);
474          constraints.add(constraint);
475        } catch (InvocationTargetException | NoSuchMethodException | ClassNotFoundException
476          | InstantiationException | IllegalAccessException e1) {
477          throw new IOException(e1);
478        }
479      }
480    }
481    // sort them, based on the priorities
482    Collections.sort(constraints, constraintComparator);
483    return constraints;
484  }
485
486  private static final Comparator<Constraint> constraintComparator = new Comparator<Constraint>() {
487    @Override
488    public int compare(Constraint c1, Constraint c2) {
489      // compare the priorities of the constraints stored in their configuration
490      return Long.compare(c1.getConf().getLong(PRIORITY_KEY, DEFAULT_PRIORITY),
491        c2.getConf().getLong(PRIORITY_KEY, DEFAULT_PRIORITY));
492    }
493  };
494
495}