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  package org.apache.hadoop.hbase.mapreduce;
20  
21  import java.io.IOException;
22  import java.lang.reflect.InvocationTargetException;
23  import java.lang.reflect.Method;
24  import java.util.ArrayList;
25  import java.util.Collections;
26  import java.util.List;
27  import java.util.Map;
28  import java.util.TreeMap;
29  import java.util.UUID;
30  
31  import org.apache.commons.logging.Log;
32  import org.apache.commons.logging.LogFactory;
33  import org.apache.hadoop.classification.InterfaceAudience;
34  import org.apache.hadoop.classification.InterfaceStability;
35  import org.apache.hadoop.conf.Configuration;
36  import org.apache.hadoop.conf.Configured;
37  import org.apache.hadoop.fs.Path;
38  import org.apache.hadoop.hbase.Cell;
39  import org.apache.hadoop.hbase.CellUtil;
40  import org.apache.hadoop.hbase.HBaseConfiguration;
41  import org.apache.hadoop.hbase.KeyValue;
42  import org.apache.hadoop.hbase.KeyValueUtil;
43  import org.apache.hadoop.hbase.ZooKeeperConnectionException;
44  import org.apache.hadoop.hbase.client.Delete;
45  import org.apache.hadoop.hbase.client.Durability;
46  import org.apache.hadoop.hbase.client.HBaseAdmin;
47  import org.apache.hadoop.hbase.client.HTable;
48  import org.apache.hadoop.hbase.client.Mutation;
49  import org.apache.hadoop.hbase.client.Put;
50  import org.apache.hadoop.hbase.client.Result;
51  import org.apache.hadoop.hbase.filter.Filter;
52  import org.apache.hadoop.hbase.io.ImmutableBytesWritable;
53  import org.apache.hadoop.hbase.util.Bytes;
54  import org.apache.hadoop.hbase.zookeeper.ZKClusterId;
55  import org.apache.hadoop.hbase.zookeeper.ZooKeeperWatcher;
56  import org.apache.hadoop.mapreduce.Job;
57  import org.apache.hadoop.mapreduce.lib.input.FileInputFormat;
58  import org.apache.hadoop.mapreduce.lib.input.SequenceFileInputFormat;
59  import org.apache.hadoop.mapreduce.lib.output.FileOutputFormat;
60  import org.apache.hadoop.util.GenericOptionsParser;
61  import org.apache.hadoop.util.Tool;
62  import org.apache.hadoop.util.ToolRunner;
63  import org.apache.zookeeper.KeeperException;
64  
65  
66  /**
67   * Import data written by {@link Export}.
68   */
69  @InterfaceAudience.Public
70  @InterfaceStability.Stable
71  public class Import extends Configured implements Tool {
72    private static final Log LOG = LogFactory.getLog(Import.class);
73    final static String NAME = "import";
74    public final static String CF_RENAME_PROP = "HBASE_IMPORTER_RENAME_CFS";
75    public final static String BULK_OUTPUT_CONF_KEY = "import.bulk.output";
76    public final static String FILTER_CLASS_CONF_KEY = "import.filter.class";
77    public final static String FILTER_ARGS_CONF_KEY = "import.filter.args";
78    public final static String TABLE_NAME = "import.table.name";
79    public final static String WAL_DURABILITY = "import.wal.durability";
80  
81    /**
82     * A mapper that just writes out KeyValues.
83     */
84    public static class KeyValueImporter extends TableMapper<ImmutableBytesWritable, KeyValue> {
85      private Map<byte[], byte[]> cfRenameMap;
86      private Filter filter;
87      private static final Log LOG = LogFactory.getLog(KeyValueImporter.class);
88  
89      /**
90       * @param row  The current table row key.
91       * @param value  The columns.
92       * @param context  The current context.
93       * @throws IOException When something is broken with the data.
94       */
95      @Override
96      public void map(ImmutableBytesWritable row, Result value,
97        Context context)
98      throws IOException {
99        try {
100         if (LOG.isTraceEnabled()) {
101           LOG.trace("Considering the row."
102               + Bytes.toString(row.get(), row.getOffset(), row.getLength()));
103         }
104         if (filter == null || !filter.filterRowKey(row.get(), row.getOffset(), row.getLength())) {
105           for (Cell kv : value.rawCells()) {
106             kv = filterKv(filter, kv);
107             // skip if we filtered it out
108             if (kv == null) continue;
109             // TODO get rid of ensureKeyValue
110             context.write(row, KeyValueUtil.ensureKeyValue(convertKv(kv, cfRenameMap)));
111           }
112         }
113       } catch (InterruptedException e) {
114         e.printStackTrace();
115       }
116     }
117 
118     @Override
119     public void setup(Context context) {
120       cfRenameMap = createCfRenameMap(context.getConfiguration());
121       filter = instantiateFilter(context.getConfiguration());
122     }
123   }
124 
125   /**
126    * Write table content out to files in hdfs.
127    */
128   public static class Importer extends TableMapper<ImmutableBytesWritable, Mutation> {
129     private Map<byte[], byte[]> cfRenameMap;
130     private List<UUID> clusterIds;
131     private Filter filter;
132     private Durability durability;
133 
134     /**
135      * @param row  The current table row key.
136      * @param value  The columns.
137      * @param context  The current context.
138      * @throws IOException When something is broken with the data.
139      */
140     @Override
141     public void map(ImmutableBytesWritable row, Result value,
142       Context context)
143     throws IOException {
144       try {
145         writeResult(row, value, context);
146       } catch (InterruptedException e) {
147         e.printStackTrace();
148       }
149     }
150 
151     private void writeResult(ImmutableBytesWritable key, Result result, Context context)
152     throws IOException, InterruptedException {
153       Put put = null;
154       Delete delete = null;
155       if (LOG.isTraceEnabled()) {
156         LOG.trace("Considering the row."
157             + Bytes.toString(key.get(), key.getOffset(), key.getLength()));
158       }
159       if (filter == null || !filter.filterRowKey(key.get(), key.getOffset(), key.getLength())) {
160         processKV(key, result, context, put, delete);
161       }
162     }
163 
164     protected void processKV(ImmutableBytesWritable key, Result result, Context context, Put put,
165         Delete delete) throws IOException, InterruptedException {
166       for (Cell kv : result.rawCells()) {
167         kv = filterKv(filter, kv);
168         // skip if we filter it out
169         if (kv == null) continue;
170 
171         kv = convertKv(kv, cfRenameMap);
172         // Deletes and Puts are gathered and written when finished
173         if (CellUtil.isDelete(kv)) {
174           if (delete == null) {
175             delete = new Delete(key.get());
176           }
177           delete.addDeleteMarker(kv);
178         } else {
179           if (put == null) {
180             put = new Put(key.get());
181           }
182           addPutToKv(put, kv);
183         }
184       }
185       if (put != null) {
186         if (durability != null) {
187           put.setDurability(durability);
188         }
189         put.setClusterIds(clusterIds);
190         context.write(key, put);
191       }
192       if (delete != null) {
193         if (durability != null) {
194           delete.setDurability(durability);
195         }
196         delete.setClusterIds(clusterIds);
197         context.write(key, delete);
198       }
199     }
200 
201     protected void addPutToKv(Put put, Cell kv) throws IOException {
202       put.add(kv);
203     }
204 
205     @Override
206     public void setup(Context context) {
207       Configuration conf = context.getConfiguration();
208       cfRenameMap = createCfRenameMap(conf);
209       filter = instantiateFilter(conf);
210       String durabilityStr = conf.get(WAL_DURABILITY);
211       if(durabilityStr != null){
212         durability = Durability.valueOf(durabilityStr.toUpperCase());
213       }
214       // TODO: This is kind of ugly doing setup of ZKW just to read the clusterid.
215       ZooKeeperWatcher zkw = null;
216       try {
217         zkw = new ZooKeeperWatcher(conf, context.getTaskAttemptID().toString(), null);
218         clusterIds = Collections.singletonList(ZKClusterId.getUUIDForCluster(zkw));
219       } catch (ZooKeeperConnectionException e) {
220         LOG.error("Problem connecting to ZooKeper during task setup", e);
221       } catch (KeeperException e) {
222         LOG.error("Problem reading ZooKeeper data during task setup", e);
223       } catch (IOException e) {
224         LOG.error("Problem setting up task", e);
225       } finally {
226         if (zkw != null) zkw.close();
227       }
228     }
229   }
230 
231   /**
232    * Create a {@link Filter} to apply to all incoming keys ({@link KeyValue KeyValues}) to
233    * optionally not include in the job output
234    * @param conf {@link Configuration} from which to load the filter
235    * @return the filter to use for the task, or <tt>null</tt> if no filter to should be used
236    * @throws IllegalArgumentException if the filter is misconfigured
237    */
238   public static Filter instantiateFilter(Configuration conf) {
239     // get the filter, if it was configured    
240     Class<? extends Filter> filterClass = conf.getClass(FILTER_CLASS_CONF_KEY, null, Filter.class);
241     if (filterClass == null) {
242       LOG.debug("No configured filter class, accepting all keyvalues.");
243       return null;
244     }
245     LOG.debug("Attempting to create filter:" + filterClass);
246     String[] filterArgs = conf.getStrings(FILTER_ARGS_CONF_KEY);
247     ArrayList<byte[]> quotedArgs = toQuotedByteArrays(filterArgs);
248     try {
249       Method m = filterClass.getMethod("createFilterFromArguments", ArrayList.class);
250       return (Filter) m.invoke(null, quotedArgs);
251     } catch (IllegalAccessException e) {
252       LOG.error("Couldn't instantiate filter!", e);
253       throw new RuntimeException(e);
254     } catch (SecurityException e) {
255       LOG.error("Couldn't instantiate filter!", e);
256       throw new RuntimeException(e);
257     } catch (NoSuchMethodException e) {
258       LOG.error("Couldn't instantiate filter!", e);
259       throw new RuntimeException(e);
260     } catch (IllegalArgumentException e) {
261       LOG.error("Couldn't instantiate filter!", e);
262       throw new RuntimeException(e);
263     } catch (InvocationTargetException e) {
264       LOG.error("Couldn't instantiate filter!", e);
265       throw new RuntimeException(e);
266     }
267   }
268 
269   private static ArrayList<byte[]> toQuotedByteArrays(String... stringArgs) {
270     ArrayList<byte[]> quotedArgs = new ArrayList<byte[]>();
271     for (String stringArg : stringArgs) {
272       // all the filters' instantiation methods expected quoted args since they are coming from
273       // the shell, so add them here, though it shouldn't really be needed :-/
274       quotedArgs.add(Bytes.toBytes("'" + stringArg + "'"));
275     }
276     return quotedArgs;
277   }
278 
279   /**
280    * Attempt to filter out the keyvalue
281    * @param kv {@link KeyValue} on which to apply the filter
282    * @return <tt>null</tt> if the key should not be written, otherwise returns the original
283    *         {@link KeyValue}
284    */
285   public static Cell filterKv(Filter filter, Cell kv) throws IOException {
286     // apply the filter and skip this kv if the filter doesn't apply
287     if (filter != null) {
288       Filter.ReturnCode code = filter.filterKeyValue(kv);
289       if (LOG.isTraceEnabled()) {
290         LOG.trace("Filter returned:" + code + " for the key value:" + kv);
291       }
292       // if its not an accept type, then skip this kv
293       if (!(code.equals(Filter.ReturnCode.INCLUDE) || code
294           .equals(Filter.ReturnCode.INCLUDE_AND_NEXT_COL))) {
295         return null;
296       }
297     }
298     return kv;
299   }
300 
301   // helper: create a new KeyValue based on CF rename map
302   private static Cell convertKv(Cell kv, Map<byte[], byte[]> cfRenameMap) {
303     if(cfRenameMap != null) {
304       // If there's a rename mapping for this CF, create a new KeyValue
305       byte[] newCfName = cfRenameMap.get(CellUtil.cloneFamily(kv));
306       if(newCfName != null) {
307           kv = new KeyValue(kv.getRowArray(), // row buffer 
308                   kv.getRowOffset(),        // row offset
309                   kv.getRowLength(),        // row length
310                   newCfName,                // CF buffer
311                   0,                        // CF offset 
312                   newCfName.length,         // CF length 
313                   kv.getQualifierArray(),   // qualifier buffer
314                   kv.getQualifierOffset(),  // qualifier offset
315                   kv.getQualifierLength(),  // qualifier length
316                   kv.getTimestamp(),        // timestamp
317                   KeyValue.Type.codeToType(kv.getTypeByte()), // KV Type
318                   kv.getValueArray(),       // value buffer 
319                   kv.getValueOffset(),      // value offset
320                   kv.getValueLength());     // value length
321       }
322     }
323     return kv;
324   }
325 
326   // helper: make a map from sourceCfName to destCfName by parsing a config key
327   private static Map<byte[], byte[]> createCfRenameMap(Configuration conf) {
328     Map<byte[], byte[]> cfRenameMap = null;
329     String allMappingsPropVal = conf.get(CF_RENAME_PROP);
330     if(allMappingsPropVal != null) {
331       // The conf value format should be sourceCf1:destCf1,sourceCf2:destCf2,...
332       String[] allMappings = allMappingsPropVal.split(",");
333       for (String mapping: allMappings) {
334         if(cfRenameMap == null) {
335             cfRenameMap = new TreeMap<byte[],byte[]>(Bytes.BYTES_COMPARATOR);
336         }
337         String [] srcAndDest = mapping.split(":");
338         if(srcAndDest.length != 2) {
339             continue;
340         }
341         cfRenameMap.put(srcAndDest[0].getBytes(), srcAndDest[1].getBytes());
342       }
343     }
344     return cfRenameMap;
345   }
346 
347   /**
348    * <p>Sets a configuration property with key {@link #CF_RENAME_PROP} in conf that tells
349    * the mapper how to rename column families.
350    * 
351    * <p>Alternately, instead of calling this function, you could set the configuration key 
352    * {@link #CF_RENAME_PROP} yourself. The value should look like 
353    * <pre>srcCf1:destCf1,srcCf2:destCf2,....</pre>. This would have the same effect on
354    * the mapper behavior.
355    * 
356    * @param conf the Configuration in which the {@link #CF_RENAME_PROP} key will be
357    *  set
358    * @param renameMap a mapping from source CF names to destination CF names
359    */
360   static public void configureCfRenaming(Configuration conf, 
361           Map<String, String> renameMap) {
362     StringBuilder sb = new StringBuilder();
363     for(Map.Entry<String,String> entry: renameMap.entrySet()) {
364       String sourceCf = entry.getKey();
365       String destCf = entry.getValue();
366 
367       if(sourceCf.contains(":") || sourceCf.contains(",") || 
368               destCf.contains(":") || destCf.contains(",")) {
369         throw new IllegalArgumentException("Illegal character in CF names: " 
370               + sourceCf + ", " + destCf);
371       }
372 
373       if(sb.length() != 0) {
374         sb.append(",");
375       }
376       sb.append(sourceCf + ":" + destCf);
377     }
378     conf.set(CF_RENAME_PROP, sb.toString());
379   }
380 
381   /**
382    * Add a Filter to be instantiated on import
383    * @param conf Configuration to update (will be passed to the job)
384    * @param clazz {@link Filter} subclass to instantiate on the server.
385    * @param filterArgs List of arguments to pass to the filter on instantiation
386    */
387   public static void addFilterAndArguments(Configuration conf, Class<? extends Filter> clazz,
388       List<String> filterArgs) throws IOException {
389     conf.set(Import.FILTER_CLASS_CONF_KEY, clazz.getName());
390     conf.setStrings(Import.FILTER_ARGS_CONF_KEY, filterArgs.toArray(new String[filterArgs.size()]));
391   }
392 
393   /**
394    * Sets up the actual job.
395    * @param conf The current configuration.
396    * @param args The command line parameters.
397    * @return The newly created job.
398    * @throws IOException When setting up the job fails.
399    */
400   public static Job createSubmittableJob(Configuration conf, String[] args)
401   throws IOException {
402     String tableName = args[0];
403     conf.set(TABLE_NAME, tableName);
404     Path inputDir = new Path(args[1]);
405     Job job = Job.getInstance(conf, NAME + "_" + tableName);
406     job.setJarByClass(Importer.class);
407     FileInputFormat.setInputPaths(job, inputDir);
408     job.setInputFormatClass(SequenceFileInputFormat.class);
409     String hfileOutPath = conf.get(BULK_OUTPUT_CONF_KEY);
410 
411     // make sure we get the filter in the jars
412     try {
413       Class<? extends Filter> filter = conf.getClass(FILTER_CLASS_CONF_KEY, null, Filter.class);
414       if (filter != null) {
415         TableMapReduceUtil.addDependencyJars(conf, filter);
416       }
417     } catch (Exception e) {
418       throw new IOException(e);
419     }
420 
421     if (hfileOutPath != null) {
422       job.setMapperClass(KeyValueImporter.class);
423       HTable table = new HTable(conf, tableName);
424       job.setReducerClass(KeyValueSortReducer.class);
425       Path outputDir = new Path(hfileOutPath);
426       FileOutputFormat.setOutputPath(job, outputDir);
427       job.setMapOutputKeyClass(ImmutableBytesWritable.class);
428       job.setMapOutputValueClass(KeyValue.class);
429       HFileOutputFormat.configureIncrementalLoad(job, table);
430       TableMapReduceUtil.addDependencyJars(job.getConfiguration(),
431           com.google.common.base.Preconditions.class);
432     } else {
433       // No reducers.  Just write straight to table.  Call initTableReducerJob
434       // because it sets up the TableOutputFormat.
435       job.setMapperClass(Importer.class);
436       TableMapReduceUtil.initTableReducerJob(tableName, null, job);
437       job.setNumReduceTasks(0);
438     }
439     return job;
440   }
441 
442   /*
443    * @param errorMsg Error message.  Can be null.
444    */
445   private static void usage(final String errorMsg) {
446     if (errorMsg != null && errorMsg.length() > 0) {
447       System.err.println("ERROR: " + errorMsg);
448     }
449     System.err.println("Usage: Import [options] <tablename> <inputdir>");
450     System.err.println("By default Import will load data directly into HBase. To instead generate");
451     System.err.println("HFiles of data to prepare for a bulk data load, pass the option:");
452     System.err.println("  -D" + BULK_OUTPUT_CONF_KEY + "=/path/for/output");
453     System.err
454         .println(" To apply a generic org.apache.hadoop.hbase.filter.Filter to the input, use");
455     System.err.println("  -D" + FILTER_CLASS_CONF_KEY + "=<name of filter class>");
456     System.err.println("  -D" + FILTER_ARGS_CONF_KEY + "=<comma separated list of args for filter");
457     System.err.println(" NOTE: The filter will be applied BEFORE doing key renames via the "
458         + CF_RENAME_PROP + " property. Futher, filters will only use the"
459         + " Filter#filterRowKey(byte[] buffer, int offset, int length) method to identify "
460         + " whether the current row needs to be ignored completely for processing and "
461         + " Filter#filterKeyValue(KeyValue) method to determine if the KeyValue should be added;"
462         + " Filter.ReturnCode#INCLUDE and #INCLUDE_AND_NEXT_COL will be considered as including"
463         + " the KeyValue.");
464     System.err.println("For performance consider the following options:\n"
465         + "  -Dmapreduce.map.speculative=false\n"
466         + "  -Dmapreduce.reduce.speculative=false\n"
467         + "  -D" + WAL_DURABILITY + "=<Used while writing data to hbase."
468             +" Allowed values are the supported durability values"
469             +" like SKIP_WAL/ASYNC_WAL/SYNC_WAL/...>");
470   }
471 
472   /**
473    * If the durability is set to {@link Durability#SKIP_WAL} and the data is imported to hbase, we
474    * need to flush all the regions of the table as the data is held in memory and is also not
475    * present in the Write Ahead Log to replay in scenarios of a crash. This method flushes all the
476    * regions of the table in the scenarios of import data to hbase with {@link Durability#SKIP_WAL}
477    */
478   public static void flushRegionsIfNecessary(Configuration conf) throws IOException,
479       InterruptedException {
480     String tableName = conf.get(TABLE_NAME);
481     HBaseAdmin hAdmin = null;
482     String durability = conf.get(WAL_DURABILITY);
483     // Need to flush if the data is written to hbase and skip wal is enabled.
484     if (conf.get(BULK_OUTPUT_CONF_KEY) == null && durability != null
485         && Durability.SKIP_WAL.name().equalsIgnoreCase(durability)) {
486       try {
487         hAdmin = new HBaseAdmin(conf);
488         hAdmin.flush(tableName);
489       } finally {
490         if (hAdmin != null) {
491           hAdmin.close();
492         }
493       }
494     }
495   }
496 
497   @Override
498   public int run(String[] args) throws Exception {
499     String[] otherArgs = new GenericOptionsParser(getConf(), args).getRemainingArgs();
500     if (otherArgs.length < 2) {
501       usage("Wrong number of arguments: " + otherArgs.length);
502       return -1;
503     }
504     String inputVersionString = System.getProperty(ResultSerialization.IMPORT_FORMAT_VER);
505     if (inputVersionString != null) {
506       getConf().set(ResultSerialization.IMPORT_FORMAT_VER, inputVersionString);
507     }
508     Job job = createSubmittableJob(getConf(), otherArgs);
509     boolean isJobSuccessful = job.waitForCompletion(true);
510     if(isJobSuccessful){
511       // Flush all the regions of the table
512       flushRegionsIfNecessary(getConf());
513     }
514     return (isJobSuccessful ? 0 : 1);
515   }
516 
517   /**
518    * Main entry point.
519    * @param args The command line parameters.
520    * @throws Exception When running the job fails.
521    */
522   public static void main(String[] args) throws Exception {
523     int errCode = ToolRunner.run(HBaseConfiguration.create(), new Import(), args);
524     System.exit(errCode);
525   }
526 
527 }