source: proiecte/HadoopJUnit/hadoop-0.20.1/src/test/org/apache/hadoop/mapreduce/lib/map/TestMultithreadedMapper.java @ 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: 3.8 KB
Line 
1/**
2 * Licensed to the Apache Software Foundation (ASF) under one
3 * or more contributor license agreements.  See the NOTICE file
4 * distributed with this work for additional information
5 * regarding copyright ownership.  The ASF licenses this file
6 * to you under the Apache License, Version 2.0 (the
7 * "License"); you may not use this file except in compliance
8 * with the License.  You may obtain a copy of the License at
9 *
10 *     http://www.apache.org/licenses/LICENSE-2.0
11 *
12 * Unless required by applicable law or agreed to in writing, software
13 * distributed under the License is distributed on an "AS IS" BASIS,
14 * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
15 * See the License for the specific language governing permissions and
16 * limitations under the License.
17 */
18
19package org.apache.hadoop.mapreduce.lib.map;
20
21import org.apache.hadoop.conf.Configuration;
22import org.apache.hadoop.fs.FileSystem;
23import org.apache.hadoop.fs.Path;
24import org.apache.hadoop.io.LongWritable;
25import org.apache.hadoop.io.Text;
26import org.apache.hadoop.mapred.HadoopTestCase;
27import org.apache.hadoop.mapreduce.*;
28import org.apache.hadoop.mapreduce.lib.input.FileInputFormat;
29import org.apache.hadoop.mapreduce.lib.output.FileOutputFormat;
30
31import java.io.DataOutputStream;
32import java.io.IOException;
33
34public class TestMultithreadedMapper extends HadoopTestCase {
35
36  public TestMultithreadedMapper() throws IOException {
37    super(HadoopTestCase.LOCAL_MR, HadoopTestCase.LOCAL_FS, 1, 1);
38  }
39
40  public void testOKRun() throws Exception {
41    run(false, false);
42  }
43
44  public void testIOExRun() throws Exception {
45    run(true, false);
46  }
47  public void testRuntimeExRun() throws Exception {
48    run(false, true);
49  }
50
51  private void run(boolean ioEx, boolean rtEx) throws Exception {
52    String localPathRoot = System.getProperty("test.build.data", "/tmp");
53    Path inDir = new Path(localPathRoot, "testing/mt/input");
54    Path outDir = new Path(localPathRoot, "testing/mt/output");
55
56
57    Configuration conf = createJobConf();
58    if (ioEx) {
59      conf.setBoolean("multithreaded.ioException", true);
60    }
61    if (rtEx) {
62      conf.setBoolean("multithreaded.runtimeException", true);
63    }
64
65    Job job = new Job(conf);
66    FileSystem fs = FileSystem.get(conf);
67    if (fs.exists(outDir)) {
68      fs.delete(outDir, true);
69    }
70    if (fs.exists(inDir)) {
71      fs.delete(inDir, true);
72    }
73    fs.mkdirs(inDir);
74    String input = "The quick brown fox\n" + "has many silly\n"
75      + "red fox sox\n";
76    DataOutputStream file = fs.create(new Path(inDir, "part-" + 0));
77    file.writeBytes(input);
78    file.close();
79
80    FileInputFormat.setInputPaths(job, inDir);
81    FileOutputFormat.setOutputPath(job, outDir);
82    job.setNumReduceTasks(1);
83    job.setJobName("mt");
84
85    job.setMapperClass(MultithreadedMapper.class);
86    MultithreadedMapper.setMapperClass(job, IDMap.class);
87    MultithreadedMapper.setNumberOfThreads(job, 2);
88    job.setReducerClass(Reducer.class);
89
90    job.waitForCompletion(true);
91
92    if (job.isSuccessful()) {
93      assertFalse(ioEx || rtEx);
94    }
95    else {
96      assertTrue(ioEx || rtEx);
97    }
98  }
99
100  public static class IDMap extends 
101      Mapper<LongWritable, Text, LongWritable, Text> {
102    private boolean ioEx = false;
103    private boolean rtEx = false;
104
105    public void setup(Context context) {
106      ioEx = context.getConfiguration().
107               getBoolean("multithreaded.ioException", false);
108      rtEx = context.getConfiguration().
109               getBoolean("multithreaded.runtimeException", false);
110    }
111
112    public void map(LongWritable key, Text value, Context context)
113        throws IOException, InterruptedException {
114      if (ioEx) {
115        throw new IOException();
116      }
117      if (rtEx) {
118        throw new RuntimeException();
119      }
120      super.map(key, value, context);
121    }
122  }
123}
Note: See TracBrowser for help on using the repository browser.