source: proiecte/HadoopJUnit/hadoop-0.20.1/src/test/org/apache/hadoop/hdfs/BenchmarkThroughput.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: 7.4 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 */
18package org.apache.hadoop.hdfs;
19
20import java.io.File;
21import java.io.FileInputStream;
22import java.io.FileOutputStream;
23import java.io.IOException;
24import java.io.InputStream;
25import java.io.OutputStream;
26
27import org.apache.commons.logging.Log;
28import org.apache.commons.logging.LogFactory;
29import org.apache.commons.logging.impl.Log4JLogger;
30import org.apache.hadoop.conf.Configuration;
31import org.apache.hadoop.conf.Configured;
32import org.apache.hadoop.fs.ChecksumFileSystem;
33import org.apache.hadoop.fs.FileSystem;
34import org.apache.hadoop.fs.LocalDirAllocator;
35import org.apache.hadoop.fs.Path;
36import org.apache.hadoop.util.Tool;
37import org.apache.hadoop.util.ToolRunner;
38
39import org.apache.log4j.Level;
40
41/**
42 * This class benchmarks the performance of the local file system, raw local
43 * file system and HDFS at reading and writing files. The user should invoke
44 * the main of this class and optionally include a repetition count.
45 */
46public class BenchmarkThroughput extends Configured implements Tool {
47
48  // the property in the config that specifies a working directory
49  private LocalDirAllocator dir;
50  private long startTime;
51  // the size of the buffer to use
52  private int BUFFER_SIZE;
53
54  private void resetMeasurements() {
55    startTime = System.currentTimeMillis();
56  }
57
58  private void printMeasurements() {
59    System.out.println(" time: " +
60                       ((System.currentTimeMillis() - startTime)/1000));
61  }
62
63  private Path writeLocalFile(String name, Configuration conf,
64                                     long total) throws IOException {
65    Path path = dir.getLocalPathForWrite(name, total, conf);
66    System.out.print("Writing " + name);
67    resetMeasurements();
68    OutputStream out = new FileOutputStream(new File(path.toString()));
69    byte[] data = new byte[BUFFER_SIZE];
70    for(long size=0; size < total; size += BUFFER_SIZE) {
71      out.write(data);
72    }
73    out.close();
74    printMeasurements();
75    return path;
76  }
77
78  private void readLocalFile(Path path,
79                                    String name,
80                                    Configuration conf) throws IOException {
81    System.out.print("Reading " + name);
82    resetMeasurements();
83    InputStream in = new FileInputStream(new File(path.toString()));
84    byte[] data = new byte[BUFFER_SIZE];
85    long size = 0;
86    while (size >= 0) {
87      size = in.read(data);
88    }
89    in.close();
90    printMeasurements();
91  }
92
93  private void writeAndReadLocalFile(String name,
94                                            Configuration conf,
95                                            long size
96                                           ) throws IOException {
97    Path f = null;
98    try {
99      f = writeLocalFile(name, conf, size);
100      readLocalFile(f, name, conf);
101    } finally {
102      if (f != null) {
103        new File(f.toString()).delete();
104      }
105    }
106  }
107
108  private Path writeFile(FileSystem fs,
109                                String name,
110                                Configuration conf,
111                                long total
112                                ) throws IOException {
113    Path f = dir.getLocalPathForWrite(name, total, conf);
114    System.out.print("Writing " + name);
115    resetMeasurements();
116    OutputStream out = fs.create(f);
117    byte[] data = new byte[BUFFER_SIZE];
118    for(long size = 0; size < total; size += BUFFER_SIZE) {
119      out.write(data);
120    }
121    out.close();
122    printMeasurements();
123    return f;
124  }
125
126  private void readFile(FileSystem fs,
127                               Path f,
128                               String name,
129                               Configuration conf
130                               ) throws IOException {
131    System.out.print("Reading " + name);
132    resetMeasurements();
133    InputStream in = fs.open(f);
134    byte[] data = new byte[BUFFER_SIZE];
135    long val = 0;
136    while (val >= 0) {
137      val = in.read(data);
138    }
139    in.close();
140    printMeasurements();
141  }
142
143  private void writeAndReadFile(FileSystem fs,
144                                       String name,
145                                       Configuration conf,
146                                       long size
147                                       ) throws IOException {
148    Path f = null;
149    try {
150      f = writeFile(fs, name, conf, size);
151      readFile(fs, f, name, conf);
152    } finally {
153      try {
154        if (f != null) {
155          fs.delete(f, true);
156        }
157      } catch (IOException ie) {
158        // IGNORE
159      }
160    }
161  }
162
163  private static void printUsage() {
164    ToolRunner.printGenericCommandUsage(System.err);
165    System.err.println("Usage: dfsthroughput [#reps]");
166    System.err.println("Config properties:\n" +
167      "  dfsthroughput.file.size:\tsize of each write/read (10GB)\n" +
168      "  dfsthroughput.buffer.size:\tbuffer size for write/read (4k)\n");
169  }
170
171  public int run(String[] args) throws IOException {
172    // silence the minidfs cluster
173    Log hadoopLog = LogFactory.getLog("org");
174    if (hadoopLog instanceof Log4JLogger) {
175      ((Log4JLogger) hadoopLog).getLogger().setLevel(Level.WARN);
176    }
177    int reps = 1;
178    if (args.length == 1) {
179      try {
180        reps = Integer.parseInt(args[0]);
181      } catch (NumberFormatException e) {
182        printUsage();
183        return -1;
184      }
185    } else if (args.length > 1) {
186      printUsage();
187      return -1;
188    }
189    Configuration conf = getConf();
190    // the size of the file to write
191    long SIZE = conf.getLong("dfsthroughput.file.size",
192        10L * 1024 * 1024 * 1024);
193    BUFFER_SIZE = conf.getInt("dfsthroughput.buffer.size", 4 * 1024);
194
195    String localDir = conf.get("mapred.temp.dir");
196    dir = new LocalDirAllocator("mapred.temp.dir");
197
198    System.setProperty("test.build.data", localDir);
199    System.out.println("Local = " + localDir);
200    ChecksumFileSystem checkedLocal = FileSystem.getLocal(conf);
201    FileSystem rawLocal = checkedLocal.getRawFileSystem();
202    for(int i=0; i < reps; ++i) {
203      writeAndReadLocalFile("local", conf, SIZE);
204      writeAndReadFile(rawLocal, "raw", conf, SIZE);
205      writeAndReadFile(checkedLocal, "checked", conf, SIZE);
206    }
207    MiniDFSCluster cluster = null;
208    try {
209      cluster = new MiniDFSCluster(conf, 1, true, new String[]{"/foo"});
210      cluster.waitActive();
211      FileSystem dfs = cluster.getFileSystem();
212      for(int i=0; i < reps; ++i) {
213        writeAndReadFile(dfs, "dfs", conf, SIZE);
214      }
215    } finally {
216      if (cluster != null) {
217        cluster.shutdown();
218        // clean up minidfs junk
219        rawLocal.delete(new Path(localDir, "dfs"), true);
220      }
221    }
222    return 0;
223  }
224
225  /**
226   * @param args
227   */
228  public static void main(String[] args) throws Exception {
229    int res = ToolRunner.run(new Configuration(),
230        new BenchmarkThroughput(), args);
231    System.exit(res);
232  }
233
234}
Note: See TracBrowser for help on using the repository browser.