mapred_tutorial
Posted onmapred_tutorial
Map/Reduce Tutorial Table of contents 1 2 3 4 5 Purpose...............................................................................................................................2 Pre-requisites......................................................................................................................2 Overview............................................................................................................................2 Inputs and Outputs............................................................................................................. 3 Example: WordCount v1.0................................................................................................ 3 5.1 5.2 5.3 Source Code...................................................................................................................3 Usage............................................................................................................................. 6 Walk-through.................................................................................................................7 Payload.......................................................................................................................... 9 Job Configuration........................................................................................................ 13 Task Execution & Environment.................................................................................. 14 Job Submission and Monitoring..................................................................................21 Job Input...................................................................................................................... 22 Job Output................................................................................................................... 23 Other Useful Features..................................................................................................25 Source Code.................................................................................................................31 Sample Runs................................................................................................................37 Highlights.................................................................................................................... 39 6 Map/Reduce - User Interfaces............................................................................................9 6.1 6.2 6.3 6.4 6.5 6.6 6.7 7 Example: WordCount v2.0.............................................................................................. 30 7.1 7.2 7.3 Copyright © 2008 The Apache Software Foundation. All rights reserved. Map/Reduce Tutorial
- Purpose This document comprehensively describes all user-facing facets of the Hadoop Map/Reduce framework and serves as a tutorial.
- Pre-requisites Ensure that Hadoop is installed, configured and is running. More details: • Hadoop Quick Start for first-time users. • Hadoop Cluster Setup for large, distributed clusters.
- Overview Hadoop Map/Reduce is a software framework for easily writing applications which process vast amounts of data (multi-terabyte data-sets) in-parallel on large clusters (thousands of nodes) of commodity hardware in a reliable, fault-tolerant manner. A Map/Reduce job usually splits the input data-set into independent chunks which are processed by the map tasks in a completely parallel manner. The framework sorts the outputs of the maps, which are then input to the reduce tasks. Typically both the input and the output of the job are stored in a file-system. The framework takes care of scheduling tasks, monitoring them and re-executes the failed tasks. Typically the compute nodes and the storage nodes are the same, that is, the Map/Reduce framework and the Hadoop Distributed File System (see HDFS Architecture ) are running on the same set of nodes. This configuration allows the framework to effectively schedule tasks on the nodes where data is already present, resulting in very high aggregate bandwidth across the cluster. The Map/Reduce framework consists of a single master JobTracker and one slave TaskTracker per cluster-node. The master is responsible for scheduling the jobs' component tasks on the slaves, monitoring them and re-executing the failed tasks. The slaves execute the tasks as directed by the master. Minimally, applications specify the input/output locations and supply map and reduce functions via implementations of appropriate interfaces and/or abstract-classes. These, and other job parameters, comprise the job configuration. The Hadoop job client then submits the job (jar/executable etc.) and configuration to the JobTracker which then assumes the responsibility of distributing the software/configuration to the slaves, scheduling tasks and monitoring them, providing status and diagnostic information to the job-client. Page 2 Copyright © 2008 The Apache Software Foundation. All rights reserved. Map/Reduce Tutorial Although the Hadoop framework is implemented in JavaTM, Map/Reduce applications need not be written in Java. • Hadoop Streaming is a utility which allows users to create and run jobs with any executables (e.g. shell utilities) as the mapper and/or the reducer. • Hadoop Pipes is a SWIG- compatible C++ API to implement Map/Reduce applications (non JNITM based).
- Inputs and Outputs
The Map/Reduce framework operates exclusively on
pairs, that is, the framework views the input to the job as a set of pairs and produces a set of pairs as the output of the job, conceivably of different types. The key and value classes have to be serializable by the framework and hence need to implement the Writable interface. Additionally, the key classes have to implement the WritableComparable interface to facilitate sorting by the framework. Input and Output types of a Map/Reduce job: (input) -> map -> -> combine -> -> reduce -> (output) - Example: WordCount v1.0 Before we jump into the details, lets walk through an example Map/Reduce application to get a flavour for how they work. WordCount is a simple application that counts the number of occurences of each word in a given input set. This works with a local-standalone, pseudo-distributed or fully-distributed Hadoop installation(see Hadoop Quick Start). 5.1. Source Code WordCount.java 1. 2. 3. 4. import java.io.IOException; import java.util./*; package org.myorg; Page 3 Copyright © 2008 The Apache Software Foundation. All rights reserved. Map/Reduce Tutorial
- public static class Map extends MapReduceBase implements Mapper
{ private final static IntWritable one = new IntWritable(1); private Text word = new Text(); public class WordCount { import org.apache.hadoop.fs.Path; import org.apache.hadoop.conf./; import org.apache.hadoop.io./; import org.apache.hadoop.mapred./; import org.apache.hadoop.util./;
- public static class Map extends MapReduceBase implements Mapper
- 18.
public void map(LongWritable key, Text value, OutputCollector
output, Reporter reporter) throws IOException { String line = value.toString(); StringTokenizer tokenizer = new StringTokenizer(line); while (tokenizer.hasMoreTokens()) { word.set(tokenizer.nextToken());
- 18.
public void map(LongWritable key, Text value, OutputCollector
- 25. output.collect(word, one); } } Page 4 Copyright © 2008 The Apache Software Foundation. All rights reserved. Map/Reduce Tutorial
- 28.
}
public static class Reduce extends MapReduceBase implements Reducer
{ public void reduce(Text key, Iterator values, OutputCollector output, Reporter reporter) throws IOException { int sum = 0; while (values.hasNext()) { sum += values.next().get(); } output.collect(key, new IntWritable(sum)); } } 29.
- 28.
}
public static class Reduce extends MapReduceBase implements Reducer
- 42. public static void main(String[] args) throws Exception { JobConf conf = new JobConf(WordCount.class); conf.setJobName("wordcount"); conf.setOutputKeyClass(Text.class); 43. conf.setOutputValueClass(IntWritable.class); 44. 45. conf.setMapperClass(Map.class); Page 5 Copyright © 2008 The Apache Software Foundation. All rights reserved. Map/Reduce Tutorial
- conf.setCombinerClass(Reduce.class); 47. conf.setReducerClass(Reduce.class); 48. 49. conf.setInputFormat(TextInputFormat.class); 50. conf.setOutputFormat(TextOutputFormat.class); 51. 52. FileInputFormat.setInputPaths(conf, new Path(args[0])); 53. FileOutputFormat.setOutputPath(conf, new Path(args[1])); 54. 55. 57. 58. 59. } JobClient.runJob(conf); }
5.2. Usage
Assuming HADOOP_HOME is the root of the installation and HADOOP_VERSION is the Hadoop version installed, compile WordCount.java and create a jar: $ mkdir wordcount_classes $ javac -classpath ${HADOOP_HOME}/hadoop-${HADOOP_VERSION}-core.jar -d wordcount_classes WordCount.java $ jar -cvf /usr/joe/wordcount.jar -C wordcount_classes/ . Assuming that: • /usr/joe/wordcount/input - input directory in HDFS
Page 6
Copyright © 2008 The Apache Software Foundation. All rights reserved.
Map/Reduce Tutorial
•
/usr/joe/wordcount/output - output directory in HDFS
Sample text-files as input: $ bin/hadoop dfs -ls /usr/joe/wordcount/input/ /usr/joe/wordcount/input/file01 /usr/joe/wordcount/input/file02 $ bin/hadoop dfs -cat /usr/joe/wordcount/input/file01 Hello World Bye World $ bin/hadoop dfs -cat /usr/joe/wordcount/input/file02 Hello Hadoop Goodbye Hadoop Run the application: $ bin/hadoop jar /usr/joe/wordcount.jar org.myorg.WordCount /usr/joe/wordcount/input /usr/joe/wordcount/output Output: $ bin/hadoop dfs -cat /usr/joe/wordcount/output/part-00000 Bye 1 Goodbye 1 Hadoop 2 Hello 2 World 2 Applications can specify a comma separated list of paths which would be present in the current working directory of the task using the option -files. The -libjars option allows applications to add jars to the classpaths of the maps and reduces. The -archives allows them to pass archives as arguments that are unzipped/unjarred and a link with name of the jar/zip are created in the current working directory of tasks. More details about the command line options are available at Hadoop Command Guide. Running wordcount example with -libjars and -files: hadoop jar hadoop-examples.jar wordcount -files cachefile.txt -libjars mylib.jar input output
5.3. Walk-through
The WordCount application is quite straight-forward. The Mapper implementation (lines 14-26), via the map method (lines 18-25), processes one line at a time, as provided by the specified TextInputFormat (line 49). It then splits the line into tokens separated by whitespaces, via the StringTokenizer, and emits a
Page 7
Copyright © 2008 The Apache Software Foundation. All rights reserved.
Map/Reduce Tutorial
key-value pair of <
, 1>. For the given sample input the first map emits: < Hello, 1> < World, 1> < Bye, 1> < World, 1> The second map emits: < Hello, 1> < Hadoop, 1> < Goodbye, 1> < Hadoop, 1> We'll learn more about the number of maps spawned for a given job, and how to control them in a fine-grained manner, a bit later in the tutorial. WordCount also specifies a combiner (line 46). Hence, the output of each map is passed through the local combiner (which is same as the Reducer as per the job configuration) for local aggregation, after being sorted on the keys. The output of the first map: < Bye, 1> < Hello, 1> < World, 2> The output of the second map: < Goodbye, 1> < Hadoop, 2> < Hello, 1> The Reducer implementation (lines 28-36), via the reduce method (lines 29-35) just sums up the values, which are the occurence counts for each key (i.e. words in this example). Thus the output of the job is: < Bye, 1> < Goodbye, 1> < Hadoop, 2> < Hello, 2> < World, 2> The run method specifies various facets of the job, such as the input/output paths (passed via the command line), key/value types, input/output formats etc., in the JobConf. It then calls the JobClient.runJob (line 55) to submit the and monitor its progress. Page 8 Copyright © 2008 The Apache Software Foundation. All rights reserved. Map/Reduce Tutorial We'll learn more about JobConf, JobClient, Tool and other interfaces and classes a bit later in the tutorial. - Map/Reduce - User Interfaces
This section provides a reasonable amount of detail on every user-facing aspect of the Map/Reduce framwork. This should help users implement, configure and tune their jobs in a fine-grained manner. However, please note that the javadoc for each class/interface remains the most comprehensive documentation available; this is only meant to be a tutorial. Let us first take the Mapper and Reducer interfaces. Applications typically implement them to provide the map and reduce methods. We will then discuss other core interfaces including JobConf, JobClient, Partitioner, OutputCollector, Reporter, InputFormat, OutputFormat, OutputCommitter and others. Finally, we will wrap up by discussing some useful features of the framework such as the DistributedCache, IsolationRunner etc.
6.1. Payload
Applications typically implement the Mapper and Reducer interfaces to provide the map and reduce methods. These form the core of the job. 6.1.1. Mapper Mapper maps input key/value pairs to a set of intermediate key/value pairs. Maps are the individual tasks that transform input records into intermediate records. The transformed intermediate records do not need to be of the same type as the input records. A given input pair may map to zero or many output pairs. The Hadoop Map/Reduce framework spawns one map task for each InputSplit generated by the InputFormat for the job. Overall, Mapper implementations are passed the JobConf for the job via the JobConfigurable.configure(JobConf) method and override it to initialize themselves. The framework then calls map(WritableComparable, Writable, OutputCollector, Reporter) for each key/value pair in the InputSplit for that task. Applications can then override the Closeable.close() method to perform any required cleanup. Output pairs do not need to be of the same types as input pairs. A given input pair may map
Page 9
Copyright © 2008 The Apache Software Foundation. All rights reserved.
Map/Reduce Tutorial
to zero or many output pairs. Output pairs are collected with calls to OutputCollector.collect(WritableComparable,Writable). Applications can use the Reporter to report progress, set application-level status messages and update Counters, or just indicate that they are alive. All intermediate values associated with a given output key are subsequently grouped by the framework, and passed to the Reducer(s) to determine the final output. Users can control the grouping by specifying a Comparator via JobConf.setOutputKeyComparatorClass(Class). The Mapper outputs are sorted and then partitioned per Reducer. The total number of partitions is the same as the number of reduce tasks for the job. Users can control which keys (and hence records) go to which Reducer by implementing a custom Partitioner. Users can optionally specify a combiner, via JobConf.setCombinerClass(Class), to perform local aggregation of the intermediate outputs, which helps to cut down the amount of data transferred from the Mapper to the Reducer. The intermediate, sorted outputs are always stored in a simple (key-len, key, value-len, value) format. Applications can control if, and how, the intermediate outputs are to be compressed and the CompressionCodec to be used via the JobConf.
6.1.1.1. How Many Maps?
The number of maps is usually driven by the total size of the inputs, that is, the total number of blocks of the input files. The right level of parallelism for maps seems to be around 10-100 maps per-node, although it has been set up to 300 maps for very cpu-light map tasks. Task setup takes awhile, so it is best if the maps take at least a minute to execute. Thus, if you expect 10TB of input data and have a blocksize of 128MB, you'll end up with 82,000 maps, unless setNumMapTasks(int) (which only provides a hint to the framework) is used to set it even higher. 6.1.2. Reducer Reducer reduces a set of intermediate values which share a key to a smaller set of values. The number of reduces for the job is set by the user via JobConf.setNumReduceTasks(int). Overall, Reducer implementations are passed the JobConf for the job via the JobConfigurable.configure(JobConf) method and can override it to initialize themselves. The
Page 10
Copyright © 2008 The Apache Software Foundation. All rights reserved.
Map/Reduce Tutorial
framework then calls reduce(WritableComparable, Iterator, OutputCollector, Reporter) method for each
pair in the grouped inputs. Applications can then override the Closeable.close() method to perform any required cleanup. Reducer has 3 primary phases: shuffle, sort and reduce. 6.1.2.1. Shuffle Input to the Reducer is the sorted output of the mappers. In this phase the framework fetches the relevant partition of the output of all the mappers, via HTTP. 6.1.2.2. Sort The framework groups Reducer inputs by keys (since different mappers may have output the same key) in this stage. The shuffle and sort phases occur simultaneously; while map-outputs are being fetched they are merged. Secondary Sort If equivalence rules for grouping the intermediate keys are required to be different from those for grouping keys before reduction, then one may specify a Comparator via JobConf.setOutputValueGroupingComparator(Class). Since JobConf.setOutputKeyComparatorClass(Class) can be used to control how intermediate keys are grouped, these can be used in conjunction to simulate secondary sort on values. 6.1.2.3. Reduce In this phase the reduce(WritableComparable, Iterator, OutputCollector, Reporter) method is called for each pair in the grouped inputs. The output of the reduce task is typically written to the FileSystem via OutputCollector.collect(WritableComparable, Writable). Applications can use the Reporter to report progress, set application-level status messages and update Counters, or just indicate that they are alive. The output of the Reducer is not sorted. 6.1.2.4. How Many Reduces? The right number of reduces seems to be 0.95 or 1.75 multiplied by ( / mapred.tasktracker.reduce.tasks.maximum). Page 11 Copyright © 2008 The Apache Software Foundation. All rights reserved. Map/Reduce Tutorial With 0.95 all of the reduces can launch immediately and start transfering map outputs as the maps finish. With 1.75 the faster nodes will finish their first round of reduces and launch a second wave of reduces doing a much better job of load balancing. Increasing the number of reduces increases the framework overhead, but increases load balancing and lowers the cost of failures. The scaling factors above are slightly less than whole numbers to reserve a few reduce slots in the framework for speculative-tasks and failed tasks. 6.1.2.5. Reducer NONE It is legal to set the number of reduce-tasks to zero if no reduction is desired. In this case the outputs of the map-tasks go directly to the FileSystem, into the output path set by setOutputPath(Path). The framework does not sort the map-outputs before writing them out to the FileSystem. 6.1.3. Partitioner Partitioner partitions the key space. Partitioner controls the partitioning of the keys of the intermediate map-outputs. The key (or a subset of the key) is used to derive the partition, typically by a hash function. The total number of partitions is the same as the number of reduce tasks for the job. Hence this controls which of the m reduce tasks the intermediate key (and hence the record) is sent to for reduction. HashPartitioner is the default Partitioner. 6.1.4. Reporter Reporter is a facility for Map/Reduce applications to report progress, set application-level status messages and update Counters. Mapper and Reducer implementations can use the Reporter to report progress or just indicate that they are alive. In scenarios where the application takes a significant amount of time to process individual key/value pairs, this is crucial since the framework might assume that the task has timed-out and kill that task. Another way to avoid this is to set the configuration parameter mapred.task.timeout to a high-enough value (or even set it to zero for no time-outs). Applications can also update Counters using the Reporter. Page 12 Copyright © 2008 The Apache Software Foundation. All rights reserved. Map/Reduce Tutorial 6.1.5. OutputCollector OutputCollector is a generalization of the facility provided by the Map/Reduce framework to collect data output by the Mapper or the Reducer (either the intermediate outputs or the output of the job). Hadoop Map/Reduce comes bundled with a library of generally useful mappers, reducers, and partitioners. 6.2. Job Configuration JobConf represents a Map/Reduce job configuration. JobConf is the primary interface for a user to describe a Map/Reduce job to the Hadoop framework for execution. The framework tries to faithfully execute the job as described by JobConf, however: • f Some configuration parameters may have been marked as final by administrators and hence cannot be altered. • While some job parameters are straight-forward to set (e.g. setNumReduceTasks(int)), other parameters interact subtly with the rest of the framework and/or job configuration and are more complex to set (e.g. setNumMapTasks(int)). JobConf is typically used to specify the Mapper, combiner (if any), Partitioner, Reducer, InputFormat, OutputFormat and OutputCommitter implementations. JobConf also indicates the set of input files (setInputPaths(JobConf, Path...) /addInputPath(JobConf, Path)) and (setInputPaths(JobConf, String) /addInputPaths(JobConf, String)) and where the output files should be written (setOutputPath(Path)). Optionally, JobConf is used to specify other advanced facets of the job such as the Comparator to be used, files to be put in the DistributedCache, whether intermediate and/or job outputs are to be compressed (and how), debugging via user-provided scripts (setMapDebugScript(String)/setReduceDebugScript(String)) , whether job tasks can be executed in a speculative manner (setMapSpeculativeExecution(boolean))/(setReduceSpeculativeExecution(boolean)) , maximum number of attempts per task (setMaxMapAttempts(int)/setMaxReduceAttempts(int)) , percentage of tasks failure which can be tolerated by the job (setMaxMapTaskFailuresPercent(int)/setMaxReduceTaskFailuresPercent(int)) etc. Of course, users can use set(String, String)/get(String, String) to set/get arbitrary parameters needed by applications. However, use the DistributedCache for large amounts of Page 13 Copyright © 2008 The Apache Software Foundation. All rights reserved. Map/Reduce Tutorial (read-only) data. 6.3. Task Execution & Environment The TaskTracker executes the Mapper/ Reducer task as a child process in a separate jvm. The child-task inherits the environment of the parent TaskTracker. The user can specify additional options to the child-jvm via the mapred.child.java.opts configuration parameter in the JobConf such as non-standard paths for the run-time linker to search shared libraries via -Djava.library.path=<> etc. If the mapred.child.java.opts contains the symbol @taskid@ it is interpolated with value of taskid of the map/reduce task. Here is an example with multiple arguments and substitutions, showing jvm GC logging, and start of a passwordless JVM JMX agent so that it can connect with jconsole and the likes to watch child memory, threads and get thread dumps. It also sets the maximum heap-size of the child jvm to 512MB and adds an additional path to the java.library.path of the child-jvm. mapred.child.java.opts -Xmx512M -Djava.library.path=/home/mycompany/lib -verbose:gc -Xloggc:/tmp/@taskid@.gc -Dcom.sun.management.jmxremote.authenticate=false -Dcom.sun.management.jmxremote.ssl=false pairs from an InputSplit. Typically the RecordReader converts the byte-oriented view of the input, provided by the InputSplit, and presents a record-oriented to the Mapper implementations for processing. RecordReader thus assumes the responsibility of processing record boundaries and presents the tasks with keys and values. 6.6. Job Output OutputFormat describes the output-specification for a Map/Reduce job. The Map/Reduce framework relies on the OutputFormat of the job to: 1. Validate the output-specification of the job; for example, check that the output directory doesn't already exist. 2. Provide the RecordWriter implementation used to write the output files of the job. Output files are stored in a FileSystem. TextOutputFormat is the default OutputFormat. 6.6.1. OutputCommitter OutputCommitter describes the commit of task output for a Map/Reduce job. The Map/Reduce framework relies on the OutputCommitter of the job to: 1. Setup the job during initialization. For example, create the temporary output directory for the job during the initialization of the job. Job setup is done by a separate task when the job is in PREP state and after initializing tasks. Once the setup task completes, the job will be moved to RUNNING state. 2. Cleanup the job after the job completion. For example, remove the temporary output directory after the job completion. Job cleanup is done by a separate task at the end of the Page 23 Copyright © 2008 The Apache Software Foundation. All rights reserved. Map/Reduce Tutorial job. Job is declared SUCCEDED/FAILED/KILLED after the cleanup task completes. 3. Setup the task temporary output. Task setup is done as part of the same task, during task initialization. 4. Check whether a task needs a commit. This is to avoid the commit procedure if a task does not need commit. 5. Commit of the task output. Once task is done, the task will commit it's output if required. 6. Discard the task commit. If the task has been failed/killed, the output will be cleaned-up. If task could not cleanup (in exception block), a separate task will be launched with same attempt-id to do the cleanup. FileOutputCommitter is the default OutputCommitter. Job setup/cleanup tasks occupy map or reduce slots, whichever is free on the TaskTracker. And JobCleanup task, TaskCleanup tasks and JobSetup task have the highest priority, and in that order. 6.6.2. Task Side-Effect Files In some applications, component tasks need to create and/or write to side-files, which differ from the actual job-output files. In such cases there could be issues with two instances of the same Mapper or Reducer running simultaneously (for example, speculative tasks) trying to open and/or write to the same file (path) on the FileSystem. Hence the application-writer will have to pick unique names per task-attempt (using the attemptid, say attempt_200709221812_0001_m_000000_0), not just per task. To avoid these issues the Map/Reduce framework, when the OutputCommitter is FileOutputCommitter, maintains a special ${mapred.output.dir}/_temporary/ ${taskid} sub-directory accessible via ${mapred.work.output.dir} for each task-attempt on the FileSystem where the output of the task-attempt is stored. On successful completion of the task-attempt, the files in the ${mapred.output.dir}/temporary/${taskid} (only) are promoted to ${mapred.output.dir}. Of course, the framework discards the sub-directory of unsuccessful task-attempts. This process is completely transparent to the application. The application-writer can take advantage of this feature by creating any side-files required in ${mapred.work.output.dir} during execution of a task via FileOutputFormat.getWorkOutputPath(), and the framework will promote them similarly for succesful task-attempts, thus eliminating the need to pick unique paths per task-attempt. Note: The value of ${mapred.work.output.dir} during execution of a particular task-attempt is actually ${mapred.output.dir}/temporary/{$taskid}, and this value is set by the Map/Reduce framework. So, just create any side-files in the path returned by FileOutputFormat.getWorkOutputPath() from map/reduce task to take advantage Page 24 Copyright © 2008 The Apache Software Foundation. All rights reserved. Map/Reduce Tutorial of this feature. The entire discussion holds true for maps of jobs with reducer=NONE (i.e. 0 reduces) since output of the map, in that case, goes directly to HDFS. 6.6.3. RecordWriter RecordWriter writes the outputpairs to an output file. RecordWriter implementations write the job outputs to the FileSystem. 6.7. Other Useful Features 6.7.1. Submitting Jobs to Queues Users submit jobs to Queues. Queues, as collection of jobs, allow the system to provide specific functionality. For example, queues use ACLs to control which users who can submit jobs to them. Queues are expected to be primarily used by Hadoop Schedulers. Hadoop comes configured with a single mandatory queue, called 'default'. Queue names are defined in the mapred.queue.names property of the Hadoop site configuration. Some job schedulers, such as the Capacity Scheduler, support multiple queues. A job defines the queue it needs to be submitted to through the mapred.job.queue.name property, or through the setQueueName(String) API. Setting the queue name is optional. If a job is submitted without an associated queue name, it is submitted to the 'default' queue. 6.7.2. Counters Counters represent global counters, defined either by the Map/Reduce framework or applications. Each Counter can be of any Enum type. Counters of a particular Enum are bunched into groups of type Counters.Group. Applications can define arbitrary Counters (of type Enum) and update them via Reporter.incrCounter(Enum, long) or Reporter.incrCounter(String, String, long) in the map and/or reduce methods. These counters are then globally aggregated by the framework. 6.7.3. DistributedCache DistributedCache distributes application-specific, large, read-only files efficiently. DistributedCache is a facility provided by the Map/Reduce framework to cache files Page 25 Copyright © 2008 The Apache Software Foundation. All rights reserved. Map/Reduce Tutorial (text, archives, jars and so on) needed by applications. Applications specify the files to be cached via urls (hdfs://) in the JobConf. The DistributedCache assumes that the files specified via hdfs:// urls are already present on the FileSystem. The framework will copy the necessary files to the slave node before any tasks for the job are executed on that node. Its efficiency stems from the fact that the files are only copied once per job and the ability to cache archives which are un-archived on the slaves. DistributedCache tracks the modification timestamps of the cached files. Clearly the cache files should not be modified by the application or externally while the job is executing. DistributedCache can be used to distribute simple, read-only data/text files and more complex types such as archives and jars. Archives (zip, tar, tgz and tar.gz files) are un-archived at the slave nodes. Files have execution permissions set. The files/archives can be distributed by setting the property mapred.cache.{files|archives}. If more than one file/archive has to be distributed, they can be added as comma separated paths. The properties can also be set by APIs DistributedCache.addCacheFile(URI,conf)/ DistributedCache.addCacheArchive(URI,conf) and DistributedCache.setCacheFiles(URIs,conf)/ DistributedCache.setCacheArchives(URIs,conf) where URI is of the form hdfs://host:port/absolute-path/#link-name. In Streaming, the files can be distributed through command line option -cacheFile/-cacheArchive. Optionally users can also direct the DistributedCache to symlink the cached file(s) into the current working directory of the task via the DistributedCache.createSymlink(Configuration) api. Or by setting the configuration property mapred.create.symlink as yes. The DistributedCache will use the fragment of the URI as the name of the symlink. For example, the URI hdfs://namenode:port/lib.so.1/#lib.so will have the symlink name as lib.so in task's cwd for the file lib.so.1 in distributed cache. The DistributedCache can also be used as a rudimentary software distribution mechanism for use in the map and/or reduce tasks. It can be used to distribute both jars and native libraries. The DistributedCache.addArchiveToClassPath(Path, Configuration) or DistributedCache.addFileToClassPath(Path, Configuration) api can be used to cache files/jars and also add them to the classpath of child-jvm. The same can be done by setting the configuration properties mapred.job.classpath.{files|archives}. Similarly the cached files that are symlinked into the working directory of the task can be used to distribute native libraries and load them. Page 26 Copyright © 2008 The Apache Software Foundation. All rights reserved. Map/Reduce Tutorial 6.7.4. Tool The Tool interface supports the handling of generic Hadoop command-line options. Tool is the standard for any Map/Reduce tool or application. The application should delegate the handling of standard command-line options to GenericOptionsParser via ToolRunner.run(Tool, String[]) and only handle its custom arguments. The generic Hadoop command-line options are: -conf -D -fs -jt 6.7.5. IsolationRunner IsolationRunner is a utility to help debug Map/Reduce programs. To use the IsolationRunner, first set keep.failed.tasks.files to true (also see keep.tasks.files.pattern). Next, go to the node on which the failed task ran and go to the TaskTracker's local directory and run the IsolationRunner: $ cd /taskTracker/${taskid}/work $ bin/hadoop org.apache.hadoop.mapred.IsolationRunner ../job.xml IsolationRunner will run the failed task in a single jvm, which can be in the debugger, over precisely the same input. 6.7.6. Profiling Profiling is a utility to get a representative (2 or 3) sample of built-in java profiler for a sample of maps and reduces. User can specify whether the system should collect profiler information for some of the tasks in the job by setting the configuration property mapred.task.profile. The value can be set using the api JobConf.setProfileEnabled(boolean). If the value is set true, the task profiling is enabled. The profiler information is stored in the user log directory. By default, profiling is not enabled for the job. Once user configures that profiling is needed, she/he can use the configuration property mapred.task.profile.{maps|reduces} to set the ranges of map/reduce tasks to Page 27 Copyright © 2008 The Apache Software Foundation. All rights reserved. Map/Reduce Tutorial profile. The value can be set using the api JobConf.setProfileTaskRange(boolean,String). By default, the specified range is 0-2. User can also specify the profiler configuration arguments by setting the configuration property mapred.task.profile.params. The value can be specified using the api JobConf.setProfileParams(String). If the string contains a %s, it will be replaced with the name of the profiling output file when the task runs. These parameters are passed to the task child JVM on the command line. The default value for the profiling parameters is -agentlib:hprof=cpu=samples,heap=sites,force=n,thread=y,verbose=n,file=%s 6.7.7. Debugging The Map/Reduce framework provides a facility to run user-provided scripts for debugging. When a map/reduce task fails, a user can run a debug script, to process task logs for example. The script is given access to the task's stdout and stderr outputs, syslog and jobconf. The output from the debug script's stdout and stderr is displayed on the console diagnostics and also as part of the job UI. In the following sections we discuss how to submit a debug script with a job. The script file needs to be distributed and submitted to the framework. 6.7.7.1. How to distribute the script file: The user needs to use DistributedCache to distribute and symlink the script file. 6.7.7.2. How to submit the script: A quick way to submit the debug script is to set values for the properties mapred.map.task.debug.script and mapred.reduce.task.debug.script, for debugging map and reduce tasks respectively. These properties can also be set by using APIs JobConf.setMapDebugScript(String) and JobConf.setReduceDebugScript(String) . In streaming mode, a debug script can be submitted with the command-line options -mapdebug and -reducedebug, for debugging map and reduce tasks respectively. The arguments to the script are the task's stdout, stderr, syslog and jobconf files. The debug command, run on the node where the map/reduce task failed, is: $script $stdout $stderr $syslog $jobconf Pipes programs have the c++ program name as a fifth argument for the command. Thus for the pipes programs the command is $script $stdout $stderr $syslog $jobconf $program Page 28 Copyright © 2008 The Apache Software Foundation. All rights reserved. Map/Reduce Tutorial 6.7.7.3. Default Behavior: For pipes, a default script is run to process core dumps under gdb, prints stack trace and gives info about running threads. 6.7.8. JobControl JobControl is a utility which encapsulates a set of Map/Reduce jobs and their dependencies. 6.7.9. Data Compression Hadoop Map/Reduce provides facilities for the application-writer to specify compression for both intermediate map-outputs and the job-outputs i.e. output of the reduces. It also comes bundled with CompressionCodec implementation for the zlib compression algorithm. The gzip file format is also supported. Hadoop also provides native implementations of the above compression codecs for reasons of both performance (zlib) and non-availability of Java libraries. More details on their usage and availability are available here. 6.7.9.1. Intermediate Outputs Applications can control compression of intermediate map-outputs via the JobConf.setCompressMapOutput(boolean) api and the CompressionCodec to be used via the JobConf.setMapOutputCompressorClass(Class) api. 6.7.9.2. Job Outputs Applications can control compression of job-outputs via the FileOutputFormat.setCompressOutput(JobConf, boolean) api and the CompressionCodec to be used can be specified via the FileOutputFormat.setOutputCompressorClass(JobConf, Class) api. If the job outputs are to be stored in the SequenceFileOutputFormat, the required SequenceFile.CompressionType (i.e. RECORD / BLOCK - defaults to RECORD) can be specified via the SequenceFileOutputFormat.setOutputCompressionType(JobConf, SequenceFile.CompressionType) api. 6.7.10. Skipping Bad Records Hadoop provides an option where a certain set of bad input records can be skipped when processing map inputs. Applications can control this feature through the SkipBadRecords Page 29 Copyright © 2008 The Apache Software Foundation. All rights reserved. Map/Reduce Tutorial class. This feature can be used when map tasks crash deterministically on certain input. This usually happens due to bugs in the map function. Usually, the user would have to fix these bugs. This is, however, not possible sometimes. The bug may be in third party libraries, for example, for which the source code is not available. In such cases, the task never completes successfully even after multiple attempts, and the job fails. With this feature, only a small portion of data surrounding the bad records is lost, which may be acceptable for some applications (those performing statistical analysis on very large data, for example). By default this feature is disabled. For enabling it, refer to SkipBadRecords.setMapperMaxSkipRecords(Configuration, long) and SkipBadRecords.setReducerMaxSkipGroups(Configuration, long). With this feature enabled, the framework gets into 'skipping mode' after a certain number of map failures. For more details, see SkipBadRecords.setAttemptsToStartSkipping(Configuration, int). In 'skipping mode', map tasks maintain the range of records being processed. To do this, the framework relies on the processed record counter. See SkipBadRecords.COUNTER_MAP_PROCESSED_RECORDS and SkipBadRecords.COUNTER_REDUCE_PROCESSED_GROUPS. This counter enables the framework to know how many records have been processed successfully, and hence, what record range caused a task to crash. On further attempts, this range of records is skipped. The number of records skipped depends on how frequently the processed record counter is incremented by the application. It is recommended that this counter be incremented after every record is processed. This may not be possible in some applications that typically batch their processing. In such cases, the framework may skip additional records surrounding the bad record. Users can control the number of skipped records through SkipBadRecords.setMapperMaxSkipRecords(Configuration, long) and SkipBadRecords.setReducerMaxSkipGroups(Configuration, long). The framework tries to narrow the range of skipped records using a binary search-like approach. The skipped range is divided into two halves and only one half gets executed. On subsequent failures, the framework figures out which half contains bad records. A task will be re-executed till the acceptable skipped value is met or all task attempts are exhausted. To increase the number of task attempts, use JobConf.setMaxMapAttempts(int) and JobConf.setMaxReduceAttempts(int). Skipped records are written to HDFS in the sequence file format, for later analysis. The location can be changed through SkipBadRecords.setSkipOutputPath(JobConf, Path). - Example: WordCount v2.0
Page 30
Copyright © 2008 The Apache Software Foundation. All rights reserved.
Map/Reduce Tutorial
Here is a more complete WordCount which uses many of the features provided by the Map/Reduce framework we discussed so far. This needs the HDFS to be up and running, especially for the DistributedCache-related features. Hence it only works with a pseudo-distributed or fully-distributed Hadoop installation.
7.1. Source Code
WordCount.java 1. 2. 3. 4. 5. 6. 7. 8. 9. 10. 11. 12. 13. 14. 15. public static class Map extends MapReduceBase implements Mapper
{ public class WordCount extends Configured implements Tool { import org.apache.hadoop.fs.Path; import org.apache.hadoop.filecache.DistributedCache; import org.apache.hadoop.conf./; import org.apache.hadoop.io./; import org.apache.hadoop.mapred./; import org.apache.hadoop.util./; import java.io./; import java.util./; package org.myorg; - static enum Counters { INPUT_WORDS } Page 31 Copyright © 2008 The Apache Software Foundation. All rights reserved. Map/Reduce Tutorial
- private boolean caseSensitive = true; private Set
patternsToSkip = new HashSet (); private final static IntWritable one = new IntWritable(1); private Text word = new Text();
- private boolean caseSensitive = true; private Set
- public void configure(JobConf job) { caseSensitive = job.getBoolean("wordcount.case.sensitive", true); inputFile = job.get("map.input.file"); private long numRecords = 0; private String inputFile;
- 32. if (job.getBoolean("wordcount.skip.patterns", false)) { Path[] patternsFiles = new Path[0]; try { patternsFiles = DistributedCache.getLocalCacheFiles(job); } catch (IOException ioe) { System.err.println("Caught
- 37. Page 32 Copyright © 2008 The Apache Software Foundation. All rights reserved. Map/Reduce Tutorial exception while getting cached files: " + StringUtils.stringifyException(ioe)); 38. 39. 40. 41. 42. 43. 44. 45. 46. 47. private void parseSkipFile(Path patternsFile) { try { BufferedReader fis = new BufferedReader(new FileReader(patternsFile.toString())); String pattern = null; while ((pattern = fis.readLine()) != null) { patternsToSkip.add(pattern); } } catch (IOException ioe) { System.err.println("Caught exception while parsing the cached file '" + patternsFile + "' : " + StringUtils.stringifyException(ioe)); } } } for (Path patternsFile : patternsFiles) { parseSkipFile(patternsFile); } } }
- 53.
- 57.
public void map(LongWritable key,
Page 33
Copyright © 2008 The Apache Software Foundation. All rights reserved.
Map/Reduce Tutorial
Text value, OutputCollector
output, Reporter reporter) throws IOException { 58. String line = (caseSensitive) ? value.toString() : value.toString().toLowerCase();
- 57.
public void map(LongWritable key,
Page 33
Copyright © 2008 The Apache Software Foundation. All rights reserved.
Map/Reduce Tutorial
Text value, OutputCollector
- ""); 62. 63. 64. 65. 66. word.set(tokenizer.nextToken()); 67. 68. reporter.incrCounter(Counters.INPUT_WORDS, 1); 69. 70. 71. 72. if ((++numRecords % 100) == 0) { reporter.setStatus("Finished processing " + numRecords + " records " + "from the input file: " + inputFile); } } } } output.collect(word, one); StringTokenizer tokenizer = new StringTokenizer(line); while (tokenizer.hasMoreTokens()) { } for (String pattern : patternsToSkip) { line = line.replaceAll(pattern,
- 75. Page 34 Copyright © 2008 The Apache Software Foundation. All rights reserved. Map/Reduce Tutorial
- public static class Reduce extends MapReduceBase implements Reducer
{ public void reduce(Text key, Iterator values, OutputCollector output, Reporter reporter) throws IOException { int sum = 0; while (values.hasNext()) { sum += values.next().get(); } output.collect(key, new IntWritable(sum)); } } 78.
- public static class Reduce extends MapReduceBase implements Reducer
- 91.
public int run(String[] args) throws Exception { JobConf conf = new JobConf(getConf(), WordCount.class); conf.setJobName("wordcount");
conf.setOutputKeyClass(Text.class); 92. conf.setOutputValueClass(IntWritable.class); 93. 94. 95. conf.setMapperClass(Map.class);
Page 35
Copyright © 2008 The Apache Software Foundation. All rights reserved.
Map/Reduce Tutorial
conf.setCombinerClass(Reduce.class); 96. conf.setReducerClass(Reduce.class); 97. 98. conf.setInputFormat(TextInputFormat.class); 99. conf.setOutputFormat(TextOutputFormat.class); 100. 101. 102. 103. 104. DistributedCache.addCacheFile(new Path(args[++i]).toUri(), conf); 105. conf.setBoolean("wordcount.skip.patterns", true); 106. 107. 108. 109. 110. 111. FileInputFormat.setInputPaths(conf, new Path(other_args.get(0))); 112. FileOutputFormat.setOutputPath(conf, new Path(other_args.get(1))); 113. } else { other_args.add(args[i]); } } List
other_args = new ArrayList (); for (int i=0; i < args.length; ++i) { if ("-skip".equals(args[i])) { Page 36 Copyright © 2008 The Apache Software Foundation. All rights reserved. Map/Reduce Tutorial
- 91.
public int run(String[] args) throws Exception { JobConf conf = new JobConf(getConf(), WordCount.class); conf.setJobName("wordcount");
conf.setOutputKeyClass(Text.class); 92. conf.setOutputValueClass(IntWritable.class); 93. 94. 95. conf.setMapperClass(Map.class);
Page 35
Copyright © 2008 The Apache Software Foundation. All rights reserved.
Map/Reduce Tutorial
conf.setCombinerClass(Reduce.class); 96. conf.setReducerClass(Reduce.class); 97. 98. conf.setInputFormat(TextInputFormat.class); 99. conf.setOutputFormat(TextOutputFormat.class); 100. 101. 102. 103. 104. DistributedCache.addCacheFile(new Path(args[++i]).toUri(), conf); 105. conf.setBoolean("wordcount.skip.patterns", true); 106. 107. 108. 109. 110. 111. FileInputFormat.setInputPaths(conf, new Path(other_args.get(0))); 112. FileOutputFormat.setOutputPath(conf, new Path(other_args.get(1))); 113. } else { other_args.add(args[i]); } } List
- 119. JobClient.runJob(conf); return 0; } public static void main(String[] args) throws Exception { int res = ToolRunner.run(new Configuration(), new WordCount(), args); System.exit(res); } }
- 123. 7.2. Sample Runs Sample text-files as input: $ bin/hadoop dfs -ls /usr/joe/wordcount/input/ /usr/joe/wordcount/input/file01 /usr/joe/wordcount/input/file02 $ bin/hadoop dfs -cat /usr/joe/wordcount/input/file01 Hello World, Bye World! $ bin/hadoop dfs -cat /usr/joe/wordcount/input/file02 Hello Hadoop, Goodbye to hadoop. Run the application: $ bin/hadoop jar /usr/joe/wordcount.jar org.myorg.WordCount /usr/joe/wordcount/input /usr/joe/wordcount/output Output: $ bin/hadoop dfs -cat /usr/joe/wordcount/output/part-00000 Bye 1 Goodbye 1 Hadoop, 1 Hello 2 Page 37 Copyright © 2008 The Apache Software Foundation. All rights reserved. Map/Reduce Tutorial World! 1 World, 1 hadoop. 1 to 1 Notice that the inputs differ from the first version we looked at, and how they affect the outputs. Now, lets plug-in a pattern-file which lists the word-patterns to be ignored, via the DistributedCache. $ hadoop dfs -cat /user/joe/wordcount/patterns.txt . \, ! to Run it again, this time with more options: $ bin/hadoop jar /usr/joe/wordcount.jar org.myorg.WordCount -Dwordcount.case.sensitive=true /usr/joe/wordcount/input /usr/joe/wordcount/output -skip /user/joe/wordcount/patterns.txt As expected, the output: $ bin/hadoop dfs -cat /usr/joe/wordcount/output/part-00000 Bye 1 Goodbye 1 Hadoop 1 Hello 2 World 2 hadoop 1 Run it once more, this time switch-off case-sensitivity: $ bin/hadoop jar /usr/joe/wordcount.jar org.myorg.WordCount -Dwordcount.case.sensitive=false /usr/joe/wordcount/input /usr/joe/wordcount/output -skip /user/joe/wordcount/patterns.txt Sure enough, the output: $ bin/hadoop dfs -cat /usr/joe/wordcount/output/part-00000 bye 1 Page 38 Copyright © 2008 The Apache Software Foundation. All rights reserved. Map/Reduce Tutorial goodbye 1 hadoop 2 hello 2 world 2 7.3. Highlights The second version of WordCount improves upon the previous one by using some features offered by the Map/Reduce framework: • Demonstrates how applications can access configuration parameters in the configure method of the Mapper (and Reducer) implementations (lines 28-43). • Demonstrates how the DistributedCache can be used to distribute read-only data needed by the jobs. Here it allows the user to specify word-patterns to skip while counting (line 104). • Demonstrates the utility of the Tool interface and the GenericOptionsParser to handle generic Hadoop command-line options (lines 87-116, 119). • Demonstrates how applications can use Counters (line 68) and how they can set application-specific status information via the Reporter instance passed to the map (and reduce) method (line 72). Java and JNI are trademarks or registered trademarks of Sun Microsystems, Inc. in the United States and other countries. Page 39 Copyright © 2008 The Apache Software Foundation. All rights reserved.