source: proiecte/HadoopJUnit/hadoop-0.20.1/src/docs/src/documentation/content/xdocs/streaming.xml @ 120

Last change on this file since 120 was 120, checked in by (none), 14 years ago

Added the mail files for the Hadoop JUNit Project

  • Property svn:executable set to *
File size: 27.5 KB
Line 
1<?xml version="1.0"?>
2<!--
3  Copyright 2002-2004 The Apache Software Foundation
4
5  Licensed under the Apache License, Version 2.0 (the "License");
6  you may not use this file except in compliance with the License.
7  You may obtain a copy of the License at
8
9      http://www.apache.org/licenses/LICENSE-2.0
10
11  Unless required by applicable law or agreed to in writing, software
12  distributed under the License is distributed on an "AS IS" BASIS,
13  WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
14  See the License for the specific language governing permissions and
15  limitations under the License.
16-->
17
18<!DOCTYPE document PUBLIC "-//APACHE//DTD Documentation V2.0//EN"
19          "http://forrest.apache.org/dtd/document-v20.dtd">
20
21
22<document>
23<header>
24<title>Hadoop Streaming</title>
25<meta name="http-equiv">Content-Type</meta>
26<meta name="content">text/html;</meta>
27<meta name="charset">utf-8</meta>
28</header>
29<body>
30<section>
31<title>Hadoop Streaming</title>
32
33<p>
34Hadoop streaming is a utility that comes with the Hadoop distribution. The utility allows you to create and run Map/Reduce jobs with any executable or script as the mapper and/or the reducer. For example:
35</p>
36<source>
37$HADOOP_HOME/bin/hadoop  jar $HADOOP_HOME/hadoop-streaming.jar \
38    -input myInputDirs \
39    -output myOutputDir \
40    -mapper /bin/cat \
41    -reducer /bin/wc
42</source>
43</section>
44
45<section>
46<title>How Does Streaming Work </title>
47<p>
48In the above example, both the mapper and the reducer are executables that read the input from stdin (line by line) and emit the output to stdout. The utility will create a Map/Reduce job, submit the job to an appropriate cluster, and monitor the progress of the job until it completes.
49</p><p>
50  When an executable is specified for mappers, each mapper task will launch the executable as a separate process when the mapper is initialized. As the mapper task runs, it converts its inputs into lines and feed the lines to the stdin of the process. In the meantime, the mapper collects the line oriented outputs from the stdout of the process and converts each line into a key/value pair, which is collected as the output of the mapper. By default, the
51  <em>prefix of a line up to the first tab character</em> is the <strong>key</strong> and the rest of the line (excluding the tab character) will be the <strong>value</strong>.
52  If there is no tab character in the line, then entire line is considered as key and the value is null. However, this can be customized, as discussed later.
53</p>
54<p>
55When an executable is specified for reducers, each reducer task will launch the executable as a separate process then the reducer is initialized. As the reducer task runs, it converts its input key/values pairs into lines and feeds the lines to the stdin of the process. In the meantime, the reducer collects the line oriented outputs from the stdout of the process, converts each line into a key/value pair, which is collected as the output of the reducer. By default, the prefix of a line up to the first tab character is the key and the rest of the line (excluding the tab character) is the value. However, this can be customized, as discussed later.
56</p><p>
57This is the basis for the communication protocol between the Map/Reduce framework and the streaming mapper/reducer.
58</p><p>
59You can supply a Java class as the mapper and/or the reducer. The above example is equivalent to:
60</p>
61<source>
62$HADOOP_HOME/bin/hadoop  jar $HADOOP_HOME/hadoop-streaming.jar \
63    -input myInputDirs \
64    -output myOutputDir \
65    -mapper org.apache.hadoop.mapred.lib.IdentityMapper \
66    -reducer /bin/wc
67</source>
68<p>User can specify <code>stream.non.zero.exit.is.failure</code> as
69<code>true</code> or <code>false</code> to make a streaming task that exits
70with a non-zero status to be <code>Failure</code> 
71or <code>Success</code> respectively. By default, streaming tasks exiting
72with non-zero status are considered to be failed tasks.</p>
73
74</section>
75
76<section>
77<title>Package Files With Job Submissions</title>
78<p>
79You can specify any executable as the mapper and/or the reducer. The executables do not need to pre-exist on the machines in the cluster; however, if they don't, you will need to use "-file" option to tell the framework to pack your executable files as a part of job submission. For example:
80</p>
81<source>
82$HADOOP_HOME/bin/hadoop  jar $HADOOP_HOME/hadoop-streaming.jar \
83    -input myInputDirs \
84    -output myOutputDir \
85    -mapper myPythonScript.py \
86    -reducer /bin/wc \
87    -file myPythonScript.py
88</source>
89<p>
90The above example specifies a user defined Python executable as the mapper. The option "-file myPythonScript.py" causes the python executable shipped to the cluster machines as a part of job submission.
91</p>
92<p>
93In addition to executable files, you can also package other auxiliary files (such as dictionaries, configuration files, etc) that may be used by the mapper and/or the reducer. For example:
94</p>
95<source>
96$HADOOP_HOME/bin/hadoop  jar $HADOOP_HOME/hadoop-streaming.jar \
97    -input myInputDirs \
98    -output myOutputDir \
99    -mapper myPythonScript.py \
100    -reducer /bin/wc \
101    -file myPythonScript.py \
102    -file myDictionary.txt
103</source>
104</section>
105
106<section>
107<title>Streaming Options and Usage </title>
108
109<section>
110<title>Mapper-Only Jobs </title>
111<p>
112Often, you may want to process input data using a map function only. To do this, simply set mapred.reduce.tasks to zero. The Map/Reduce framework will not create any reducer tasks. Rather, the outputs of the mapper tasks will be the final output of the job.
113</p><p>
114To be backward compatible, Hadoop Streaming also supports the "-reduce NONE" option, which is equivalent to "-D mapred.reduce.tasks=0".
115</p>
116</section>
117
118<section>
119<title>Specifying Other Plugins for Jobs </title>
120<p>
121Just as with a normal Map/Reduce job, you can specify other plugins for a streaming job:
122</p>
123<source>
124   -inputformat JavaClassName
125   -outputformat JavaClassName
126   -partitioner JavaClassName
127   -combiner JavaClassName
128</source>
129<p>
130The class you supply for the input format should return key/value pairs of Text class. If you do not specify an input format class, the TextInputFormat is used as the default. Since the TextInputFormat returns keys of LongWritable class, which are actually not part of the input data, the keys will be discarded; only the values will be piped to the streaming mapper.
131</p><p>
132The class you supply for the output format is expected to take key/value pairs of Text class. If you do not specify an output format class, the TextOutputFormat is used as the default.
133</p>
134</section>
135
136<section>
137<title>Large files and archives in Hadoop Streaming </title>
138
139<p>
140The -files and -archives options allow you to make files and archives available to the tasks. The argument is a URI to the file or archive that you have already uploaded to HDFS. These files and archives are cached across jobs. You can retrieve the host and fs_port values from the fs.default.name config variable.
141</p>
142<p>
143Here are examples of the -files option:
144</p> 
145<source>
146-files hdfs://host:fs_port/user/testfile.txt#testlink
147</source>
148<p>
149In the above example, the part of the url after # is used as the symlink name that is created in the current working directory of tasks. So the tasks will have a symlink called testlink in the cwd that points to a local copy of testfile.txt. Multiple entries can be specified as:
150</p>
151<source>
152-files hdfs://host:fs_port/user/testfile1.txt#testlink1 -files hdfs://host:fs_port/user/testfile2.txt#testlink2
153</source>
154<p>
155The -archives option allows you to copy jars locally to the cwd of tasks and automatically unjar the files. For example:
156</p>
157<source>
158-archives hdfs://host:fs_port/user/testfile.jar#testlink3
159</source>
160<p>
161In the example above, a symlink testlink3 is created in the current working directory of tasks. This symlink points to the directory that stores the unjarred contents of the uploaded jar file.
162</p>
163<p>
164Here's another example of the -archives option. Here, the input.txt file has two lines specifying the names of the two files: testlink/cache.txt and testlink/cache2.txt. "testlink" is a symlink to the archived directory, which has the files "cache.txt" and "cache2.txt".
165</p>
166<source>
167$HADOOP_HOME/bin/hadoop  jar $HADOOP_HOME/hadoop-streaming.jar \
168                  -input "/user/me/samples/cachefile/input.txt"  \
169                  -mapper "xargs cat"  \
170                  -reducer "cat"  \
171                  -output "/user/me/samples/cachefile/out" \ 
172                  -archives 'hdfs://hadoop-nn1.example.com/user/me/samples/cachefile/cachedir.jar#testlink' \ 
173                  -D mapred.map.tasks=1 \
174                  -D mapred.reduce.tasks=1 \
175                  -D mapred.job.name="Experiment"
176
177$ ls test_jar/
178cache.txt  cache2.txt
179
180$ jar cvf cachedir.jar -C test_jar/ .
181added manifest
182adding: cache.txt(in = 30) (out= 29)(deflated 3%)
183adding: cache2.txt(in = 37) (out= 35)(deflated 5%)
184
185$ hadoop dfs -put cachedir.jar samples/cachefile
186
187$ hadoop dfs -cat /user/me/samples/cachefile/input.txt
188testlink/cache.txt
189testlink/cache2.txt
190
191$ cat test_jar/cache.txt
192This is just the cache string
193
194$ cat test_jar/cache2.txt
195This is just the second cache string
196
197$ hadoop dfs -ls /user/me/samples/cachefile/out     
198Found 1 items
199/user/me/samples/cachefile/out/part-00000  &lt;r 3&gt;   69
200
201$ hadoop dfs -cat /user/me/samples/cachefile/out/part-00000
202This is just the cache string   
203This is just the second cache string
204
205</source>
206</section>
207
208<section>
209<title>Specifying Additional Configuration Variables for Jobs </title>
210<p>
211You can specify additional configuration variables by using "-D  &lt;n&gt;=&lt;v&gt;". For example:
212</p>
213<source>
214$HADOOP_HOME/bin/hadoop  jar $HADOOP_HOME/hadoop-streaming.jar \
215    -input myInputDirs \
216    -output myOutputDir \
217    -mapper org.apache.hadoop.mapred.lib.IdentityMapper\
218    -reducer /bin/wc \
219    -D mapred.reduce.tasks=2
220</source>
221<p>
222The -D mapred.reduce.tasks=2 in the above example specifies to use two reducers for the job.
223</p>
224<p>
225For more details on the jobconf parameters see:
226<a href="ext:mapred-default">mapred-default.html</a></p>
227</section>
228
229<section>
230<title>Other Supported Options </title>
231<p>
232Other options you may specify for a streaming job are described here:
233</p>
234<table>
235<tr><th>Parameter</th><th>Optional/Required </th><th>Description </th></tr>
236
237<tr><td> -cmdenv   name=value </td><td> Optional </td><td> Pass env var to streaming commands </td></tr>
238
239<tr><td> -inputreader JavaClassName </td><td> Optional </td><td> For backwards-compatibility: specifies a record reader class (instead of an input format class) </td></tr>
240<tr><td> -verbose </td><td> Optional </td><td> Verbose output </td></tr>
241</table>
242<p>
243Streaming support Hadoop generic command line options.
244
245Supported parameters are :
246The general command line syntax is :
247<br/>    bin/hadoop command [genericOptions] [commandOptions]
248</p>
249
250<table>
251<tr><th>Parameter</th><th>Optional/Required </th><th>Description </th></tr>
252
253<tr><td> -conf  configuration_file </td><td> Optional </td><td> specify an application configuration file </td></tr>
254<tr><td> -D  property=value </td><td> Optional </td><td> use value for given property </td></tr>
255<tr><td> -fs host:port or local </td><td> Optional </td><td> specify a namenode </td></tr>
256<tr><td> -jt host:port or local </td><td> Optional </td><td> specify a job tracker </td></tr>
257<tr><td> -files </td><td> Optional </td><td> specify comma separated files to be copied to the map reduce cluster </td></tr>
258<tr><td> -archives </td><td> Optional </td><td> specify comma separated archives to be unarchived on the compute machines </td></tr>
259<tr><td>  </td><td> Optional </td><td>  </td></tr>
260<tr><td> -jt host:port or local </td><td> Optional </td><td>  </td></tr>
261</table>
262
263<p>
264To change the local temp directory use:
265</p>
266<source>
267  -D dfs.data.dir=/tmp
268</source>
269<p>
270To specify additional local temp directories use:
271</p>
272<source>
273   -D mapred.local.dir=/tmp/local
274   -D mapred.system.dir=/tmp/system
275   -D mapred.temp.dir=/tmp/temp
276</source>
277<p>
278For more details on jobconf parameters see:
279<a href="ext:mapred-default">mapred-default.html</a></p>
280<p>
281To set an environment variable in a streaming command use:
282</p>
283<source>
284-cmdenv EXAMPLE_DIR=/home/example/dictionaries/
285</source>
286</section>
287</section>
288
289<section>
290<title>More usage examples </title>
291
292<section>
293<title>Customizing the Way to Split Lines into Key/Value Pairs </title>
294<p>
295As noted earlier, when the Map/Reduce framework reads a line from the stdout of the mapper, it splits the line into a key/value pair. By default, the prefix of the line up to the first tab character is the key and the rest of the line (excluding the tab character) is the value.
296</p>
297<p>
298However, you can customize this default. You can specify a field separator other than the tab character (the default), and you can specify the nth (n >= 1) character rather than the first character in a line (the default) as the separator between the key and value. For example:
299</p>
300
301<source>
302$HADOOP_HOME/bin/hadoop  jar $HADOOP_HOME/hadoop-streaming.jar \
303    -input myInputDirs \
304    -output myOutputDir \
305    -mapper org.apache.hadoop.mapred.lib.IdentityMapper \
306    -reducer org.apache.hadoop.mapred.lib.IdentityReducer \
307    -D stream.map.output.field.separator=. \
308    -D stream.num.map.output.key.fields=4
309</source>
310<p>
311In the above example, "-D stream.map.output.field.separator=." specifies "." as the field separator for the map outputs, and the prefix up to the fourth "." in a line will be the key and the rest of the line (excluding the fourth ".") will be the value. If a line has less than four "."s, then the whole line will be the key and the value will be an empty Text object (like the one created by new Text("")).
312</p><p>
313Similarly, you can use "-D stream.reduce.output.field.separator=SEP" and "-D stream.num.reduce.output.fields=NUM" to specify the nth field separator in a line of the reduce outputs as the separator between the key and the value.
314</p>
315<p> Similarly, you can specify "stream.map.input.field.separator" and
316"stream.reduce.input.field.separator" as the input separator for map/reduce
317inputs. By default the separator is the tab character.</p>
318</section>
319
320
321<section>
322<title>A Useful Partitioner Class (secondary sort, the -partitioner org.apache.hadoop.mapred.lib.KeyFieldBasedPartitioner option) </title>
323<p>
324Hadoop has a library class,
325<a href="ext:api/org/apache/hadoop/mapred/lib/keyfieldbasedpartitioner">KeyFieldBasedPartitioner</a>,
326that is useful for many applications. This class allows the Map/Reduce
327framework to partition the map outputs based on certain key fields, not
328the whole keys. For example:
329</p>
330<source>
331$HADOOP_HOME/bin/hadoop  jar $HADOOP_HOME/hadoop-streaming.jar \
332    -input myInputDirs \
333    -output myOutputDir \
334    -mapper org.apache.hadoop.mapred.lib.IdentityMapper \
335    -reducer org.apache.hadoop.mapred.lib.IdentityReducer \
336    -partitioner org.apache.hadoop.mapred.lib.KeyFieldBasedPartitioner \
337    -D stream.map.output.field.separator=. \
338    -D stream.num.map.output.key.fields=4 \
339    -D map.output.key.field.separator=. \
340    -D mapred.text.key.partitioner.options=-k1,2\
341    -D mapred.reduce.tasks=12
342</source>
343<p>
344Here, <em>-D stream.map.output.field.separator=.</em> and <em>-D stream.num.map.output.key.fields=4</em> are as explained in previous example. The two variables are used by streaming to identify the key/value pair of mapper.
345</p><p>
346The map output keys of the above Map/Reduce job normally have four fields
347separated by ".". However, the Map/Reduce framework will partition the map
348outputs by the first two fields of the keys using the
349<em>-D mapred.text.key.partitioner.options=-k1,2</em> option.
350Here, <em>-D map.output.key.field.separator=.</em> specifies the separator
351for the partition. This guarantees that all the key/value pairs with the
352same first two fields in the keys will be partitioned into the same reducer.
353</p><p>
354<em>This is effectively equivalent to specifying the first two fields as the primary key and the next two fields as the secondary. The primary key is used for partitioning, and the combination of the primary and secondary keys is used for sorting.</em> A simple illustration is shown here:
355</p>
356<p>
357Output of map (the keys)</p><source>
35811.12.1.2
35911.14.2.3
36011.11.4.1
36111.12.1.1
36211.14.2.2
363
364</source>
365<p>
366Partition into 3 reducers (the first 2 fields are used as keys for partition)</p><source>
36711.11.4.1
368-----------
36911.12.1.2
37011.12.1.1
371-----------
37211.14.2.3
37311.14.2.2
374</source>
375<p>
376Sorting within each partition for the reducer(all 4 fields used for sorting)</p><source>
37711.11.4.1
378-----------
37911.12.1.1
38011.12.1.2
381-----------
38211.14.2.2
38311.14.2.3
384</source>
385</section>
386<section>
387<title>A Useful Comparator Class</title>
388<p>
389Hadoop has a library class,
390<a href="ext:api/org/apache/hadoop/mapred/lib/keyfieldbasedcomparator">KeyFieldBasedComparator</a>,
391that is useful for many applications. This class provides a subset of features
392provided by the Unix/GNU Sort. For example:
393</p>
394<source>
395$HADOOP_HOME/bin/hadoop  jar $HADOOP_HOME/hadoop-streaming.jar \
396    -input myInputDirs \
397    -output myOutputDir \
398    -mapper org.apache.hadoop.mapred.lib.IdentityMapper \
399    -reducer org.apache.hadoop.mapred.lib.IdentityReducer \
400    -D mapred.output.key.comparator.class=org.apache.hadoop.mapred.lib.KeyFieldBasedComparator \
401    -D stream.map.output.field.separator=. \
402    -D stream.num.map.output.key.fields=4 \
403    -D map.output.key.field.separator=. \
404    -D mapred.text.key.comparator.options=-k2,2nr\
405    -D mapred.reduce.tasks=12
406</source>
407<p>
408The map output keys of the above Map/Reduce job normally have four fields
409separated by ".". However, the Map/Reduce framework will sort the
410outputs by the second field of the keys using the
411<em>-D mapred.text.key.comparator.options=-k2,2nr</em> option.
412Here, <em>-n</em> specifies that the sorting is numerical sorting and
413<em>-r</em> specifies that the result should be reversed. A simple illustration
414is shown below:
415</p>
416<p>
417Output of map (the keys)</p>
418<source>
41911.12.1.2
42011.14.2.3
42111.11.4.1
42211.12.1.1
42311.14.2.2
424</source>
425<p>
426Sorting output for the reducer(where second field used for sorting)</p>
427<source>
42811.14.2.3
42911.14.2.2
43011.12.1.2
43111.12.1.1
43211.11.4.1
433</source>
434</section>
435
436<section>
437<title>Working with the Hadoop Aggregate Package (the -reduce aggregate option) </title>
438<p>
439Hadoop has a library package called
440<a href="ext:api/org/apache/hadoop/mapred/lib/aggregate/package-summary">Aggregate</a>.
441Aggregate provides a special reducer class and a special combiner class, and
442a list of simple aggregators that perform aggregations such as "sum", "max",
443"min" and so on  over a sequence of values. Aggregate allows you to define a
444mapper plugin class that is expected to generate "aggregatable items" for each
445input key/value pair of the mappers. The combiner/reducer will aggregate those
446aggregatable items by invoking the appropriate aggregators.
447</p><p>
448To use Aggregate, simply specify "-reducer aggregate":
449</p>
450<source>
451$HADOOP_HOME/bin/hadoop  jar $HADOOP_HOME/hadoop-streaming.jar \
452    -input myInputDirs \
453    -output myOutputDir \
454    -mapper myAggregatorForKeyCount.py \
455    -reducer aggregate \
456    -file myAggregatorForKeyCount.py \
457    -D mapred.reduce.tasks=12
458</source>
459<p>
460The python program myAggregatorForKeyCount.py looks like:
461</p>
462<source>
463#!/usr/bin/python
464
465import sys;
466
467def generateLongCountToken(id):
468    return "LongValueSum:" + id + "\t" + "1"
469
470def main(argv):
471    line = sys.stdin.readline();
472    try:
473        while line:
474            line = line&#91;:-1];
475            fields = line.split("\t");
476            print generateLongCountToken(fields&#91;0]);
477            line = sys.stdin.readline();
478    except "end of file":
479        return None
480if __name__ == "__main__":
481     main(sys.argv)
482</source>
483</section>
484
485<section>
486<title>Field Selection ( similar to unix 'cut' command) </title>
487<p>
488Hadoop has a library class, org.apache.hadoop.mapred.lib.FieldSelectionMapReduce, that effectively allows you to process text data like the unix "cut" utility. The map function defined in the class treats each input key/value pair as a list of fields. You can specify the field separator (the default is the tab character). You can select an arbitrary list of fields as the map output key, and an arbitrary list of fields as the map output value. Similarly, the reduce function defined in the class treats each input key/value pair as a list of fields. You can select an arbitrary list of fields as the reduce output key, and an arbitrary list of fields as the reduce output value. For example:
489</p>
490<source>
491$HADOOP_HOME/bin/hadoop  jar $HADOOP_HOME/hadoop-streaming.jar \
492    -input myInputDirs \
493    -output myOutputDir \
494    -mapper org.apache.hadoop.mapred.lib.FieldSelectionMapReduce\
495    -reducer org.apache.hadoop.mapred.lib.FieldSelectionMapReduce\
496    -partitioner org.apache.hadoop.mapred.lib.KeyFieldBasedPartitioner \
497    -D map.output.key.field.separa=. \
498    -D mapred.text.key.partitioner.options=-k1,2 \
499    -D mapred.data.field.separator=. \
500    -D map.output.key.value.fields.spec=6,5,1-3:0- \
501    -D reduce.output.key.value.fields.spec=0-2:5- \
502    -D mapred.reduce.tasks=12
503</source>
504<p>
505The option "-D map.output.key.value.fields.spec=6,5,1-3:0-" specifies key/value selection for the map outputs. Key selection spec and value selection spec are separated by ":". In this case, the map output key will consist of fields 6, 5, 1, 2, and 3. The map output value will consist of all fields (0- means field 0 and all
506the subsequent fields).
507</p><p>
508The option "-D reduce.output.key.value.fields.spec=0-2:5-" specifies
509key/value selection for the reduce outputs. In this case, the reduce
510output key will consist of fields 0, 1, 2 (corresponding to the original
511fields 6, 5, 1). The reduce output value will consist of all fields starting
512from field 5 (corresponding to all the original fields). 
513</p>
514</section>
515</section>
516
517<section>
518<title>Frequently Asked Questions </title>
519
520<section>
521<title>How do I use Hadoop Streaming to run an arbitrary set of (semi-)independent tasks? </title>
522<p>
523Often you do not need the full power of Map Reduce, but only need to run multiple instances of the same program - either on different parts of the data, or on the same data, but with different parameters. You can use Hadoop Streaming to do this.
524</p>
525
526</section>
527
528<section>
529<title>How do I process files, one per map? </title>
530<p>
531As an example, consider the problem of zipping (compressing) a set of files across the hadoop cluster. You can achieve this using either of these methods:
532</p><ol>
533<li> Hadoop Streaming and custom mapper script:<ul>
534  <li> Generate a file containing the full HDFS path of the input files. Each map task would get one file name as input.</li>
535  <li> Create a mapper script which, given a filename, will get the file to local disk, gzip the file and put it back in the desired output directory</li>
536</ul></li>
537<li>The existing Hadoop Framework:<ul>
538   <li>Add these commands to your main function:
539<source>
540       FileOutputFormat.setCompressOutput(conf, true);
541       FileOutputFormat.setOutputCompressorClass(conf, org.apache.hadoop.io.compress.GzipCodec.class);
542       conf.setOutputFormat(NonSplitableTextInputFormat.class);
543       conf.setNumReduceTasks(0);
544</source></li>
545   <li>Write your map function:
546<source>
547
548       public void map(WritableComparable key, Writable value,
549                               OutputCollector output,
550                               Reporter reporter) throws IOException {
551            output.collect((Text)value, null);
552       }
553</source></li>
554  <li>Note that the output filename will not be the same as the original filename</li>
555</ul></li>
556</ol>
557</section>
558
559<section>
560<title>How many reducers should I use? </title>
561<p>
562See the Hadoop Wiki for details: <a href="mapred_tutorial.html#Reducer">Reducer</a>
563</p>
564</section>
565
566<section>
567<title>If I set up an alias in my shell script, will that work after -mapper, i.e. say I do: alias c1='cut -f1'. Will -mapper "c1" work? </title>
568<p>
569Using an alias will not work, but variable substitution is allowed as shown in this example:
570</p>
571<source>
572$ hadoop dfs -cat samples/student_marks
573alice   50
574bruce   70
575charlie 80
576dan     75
577
578$ c2='cut -f2'; $HADOOP_HOME/bin/hadoop jar $HADOOP_HOME/hadoop-streaming.jar \
579    -input /user/me/samples/student_marks
580    -mapper \"$c2\" -reducer 'cat' 
581    -output /user/me/samples/student_out
582    -D mapred.job.name='Experiment'
583
584$ hadoop dfs -ls samples/student_out
585Found 1 items/user/me/samples/student_out/part-00000    &lt;r 3&gt;   16
586
587$ hadoop dfs -cat samples/student_out/part-00000
58850
58970
59075
59180
592</source>
593</section>
594
595<section>
596<title>Can I use UNIX pipes? For example, will -mapper "cut -f1 | sed s/foo/bar/g" work?</title>
597<p>
598Currently this does not work and gives an "java.io.IOException: Broken pipe" error. This is probably a bug that needs to be investigated.
599</p>
600</section>
601
602<section>
603<title>When I run a streaming job by <strong>distributing large executables</strong> (for example, 3.6G) through the -file option, I get a "No space left on device" error. What do I do? </title>
604<p>
605The jar packaging happens in a directory pointed to by the configuration variable stream.tmpdir. The default value of stream.tmpdir is /tmp. Set the value to a directory with more space:
606</p>
607<source>
608-D stream.tmpdir=/export/bigspace/...
609</source>
610</section>
611
612<section>
613<title>How do I specify multiple input directories? </title>
614<p>
615You can specify multiple input directories with multiple '-input' options:
616</p><source>
617 hadoop jar hadoop-streaming.jar -input '/user/foo/dir1' -input '/user/foo/dir2'
618</source>
619</section>
620
621<section>
622<title>How do I generate output files with gzip format? </title>
623<p>
624Instead of plain text files, you can generate gzip files as your generated output. Pass '-D mapred.output.compress=true -D  mapred.output.compression.codec=org.apache.hadoop.io.compress.GzipCode' as option to your streaming job.
625</p>
626</section>
627
628<section>
629<title>How do I provide my own input/output format with streaming? </title>
630<p>
631At least as late as version 0.14, Hadoop does not support multiple jar files. So, when specifying your own custom classes you will have to pack them along with the streaming jar and use the custom jar instead of the default hadoop streaming jar.
632</p>
633</section>
634
635<section>
636<title>How do I parse XML documents using streaming? </title>
637<p>
638You can use the record reader StreamXmlRecordReader to process XML documents.
639</p>
640<source>
641hadoop jar hadoop-streaming.jar -inputreader "StreamXmlRecord,begin=BEGIN_STRING,end=END_STRING" ..... (rest of the command)
642</source>
643<p>
644Anything found between BEGIN_STRING and END_STRING would be treated as one record for map tasks.
645</p>
646</section>
647
648<section>
649<title>How do I update counters in streaming applications? </title>
650<p>
651A streaming process can use the stderr to emit counter information.
652<code>reporter:counter:&lt;group&gt;,&lt;counter&gt;,&lt;amount&gt;</code> 
653should be sent to stderr to update the counter.
654</p>
655</section>
656
657<section>
658<title>How do I update status in streaming applications? </title>
659<p>
660A streaming process can use the stderr to emit status information.
661To set a status, <code>reporter:status:&lt;message&gt;</code> should be sent
662to stderr.
663</p>
664</section>
665
666</section>
667</body>
668</document>
Note: See TracBrowser for help on using the repository browser.