source: proiecte/HadoopJUnit/hadoop-0.20.1/src/test/org/apache/hadoop/hdfs/TestFileAppend3.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: 9.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.IOException;
21import java.io.RandomAccessFile;
22
23import junit.extensions.TestSetup;
24import junit.framework.Test;
25import junit.framework.TestSuite;
26
27import org.apache.hadoop.conf.Configuration;
28import org.apache.hadoop.fs.FSDataOutputStream;
29import org.apache.hadoop.fs.Path;
30import org.apache.hadoop.hdfs.protocol.Block;
31import org.apache.hadoop.hdfs.protocol.DatanodeInfo;
32import org.apache.hadoop.hdfs.protocol.LocatedBlock;
33import org.apache.hadoop.hdfs.protocol.LocatedBlocks;
34import org.apache.hadoop.hdfs.server.datanode.DataNode;
35import org.apache.hadoop.hdfs.server.datanode.FSDataset;
36import org.apache.hadoop.hdfs.server.protocol.BlockMetaDataInfo;
37
38/** This class implements some of tests posted in HADOOP-2658. */
39public class TestFileAppend3 extends junit.framework.TestCase {
40  static final long BLOCK_SIZE = 64 * 1024;
41  static final short REPLICATION = 3;
42  static final int DATANODE_NUM = 5;
43
44  private static Configuration conf;
45  private static int buffersize;
46  private static MiniDFSCluster cluster;
47  private static DistributedFileSystem fs;
48
49  public static Test suite() {
50    return new TestSetup(new TestSuite(TestFileAppend3.class)) {
51      protected void setUp() throws java.lang.Exception {
52        AppendTestUtil.LOG.info("setUp()");
53        conf = new Configuration();
54        conf.setInt("io.bytes.per.checksum", 512);
55        conf.setBoolean("dfs.support.append", true);
56        buffersize = conf.getInt("io.file.buffer.size", 4096);
57        cluster = new MiniDFSCluster(conf, DATANODE_NUM, true, null);
58        fs = (DistributedFileSystem)cluster.getFileSystem();
59      }
60   
61      protected void tearDown() throws Exception {
62        AppendTestUtil.LOG.info("tearDown()");
63        if(fs != null) fs.close();
64        if(cluster != null) cluster.shutdown();
65      }
66    }; 
67  }
68
69  /** TC1: Append on block boundary. */
70  public void testTC1() throws Exception {
71    final Path p = new Path("/TC1/foo");
72    System.out.println("p=" + p);
73
74    //a. Create file and write one block of data. Close file.
75    final int len1 = (int)BLOCK_SIZE; 
76    {
77      FSDataOutputStream out = fs.create(p, false, buffersize, REPLICATION, BLOCK_SIZE);
78      AppendTestUtil.write(out, 0, len1);
79      out.close();
80    }
81
82    //   Reopen file to append. Append half block of data. Close file.
83    final int len2 = (int)BLOCK_SIZE/2; 
84    {
85      FSDataOutputStream out = fs.append(p);
86      AppendTestUtil.write(out, len1, len2);
87      out.close();
88    }
89   
90    //b. Reopen file and read 1.5 blocks worth of data. Close file.
91    AppendTestUtil.check(fs, p, len1 + len2);
92  }
93
94  /** TC2: Append on non-block boundary. */
95  public void testTC2() throws Exception {
96    final Path p = new Path("/TC2/foo");
97    System.out.println("p=" + p);
98
99    //a. Create file with one and a half block of data. Close file.
100    final int len1 = (int)(BLOCK_SIZE + BLOCK_SIZE/2); 
101    {
102      FSDataOutputStream out = fs.create(p, false, buffersize, REPLICATION, BLOCK_SIZE);
103      AppendTestUtil.write(out, 0, len1);
104      out.close();
105    }
106
107    //   Reopen file to append quarter block of data. Close file.
108    final int len2 = (int)BLOCK_SIZE/4; 
109    {
110      FSDataOutputStream out = fs.append(p);
111      AppendTestUtil.write(out, len1, len2);
112      out.close();
113    }
114
115    //b. Reopen file and read 1.75 blocks of data. Close file.
116    AppendTestUtil.check(fs, p, len1 + len2);
117  }
118
119  /** TC5: Only one simultaneous append. */
120  public void testTC5() throws Exception {
121    final Path p = new Path("/TC5/foo");
122    System.out.println("p=" + p);
123
124    //a. Create file on Machine M1. Write half block to it. Close file.
125    {
126      FSDataOutputStream out = fs.create(p, false, buffersize, REPLICATION, BLOCK_SIZE);
127      AppendTestUtil.write(out, 0, (int)(BLOCK_SIZE/2));
128      out.close();
129    }
130
131    //b. Reopen file in "append" mode on Machine M1.
132    FSDataOutputStream out = fs.append(p);
133
134    //c. On Machine M2, reopen file in "append" mode. This should fail.
135    try {
136      AppendTestUtil.createHdfsWithDifferentUsername(conf).append(p);
137      fail("This should fail.");
138    } catch(IOException ioe) {
139      AppendTestUtil.LOG.info("GOOD: got an exception", ioe);
140    }
141
142    //d. On Machine M1, close file.
143    out.close();       
144  }
145
146  /** TC7: Corrupted replicas are present. */
147  public void testTC7() throws Exception {
148    final short repl = 2;
149    final Path p = new Path("/TC7/foo");
150    System.out.println("p=" + p);
151   
152    //a. Create file with replication factor of 2. Write half block of data. Close file.
153    final int len1 = (int)(BLOCK_SIZE/2); 
154    {
155      FSDataOutputStream out = fs.create(p, false, buffersize, repl, BLOCK_SIZE);
156      AppendTestUtil.write(out, 0, len1);
157      out.close();
158    }
159    DFSTestUtil.waitReplication(fs, p, repl);
160
161    //b. Log into one datanode that has one replica of this block.
162    //   Find the block file on this datanode and truncate it to zero size.
163    final LocatedBlocks locatedblocks = fs.dfs.namenode.getBlockLocations(p.toString(), 0L, len1);
164    assertEquals(1, locatedblocks.locatedBlockCount());
165    final LocatedBlock lb = locatedblocks.get(0);
166    final Block blk = lb.getBlock();
167    assertEquals(len1, lb.getBlockSize());
168
169    DatanodeInfo[] datanodeinfos = lb.getLocations();
170    assertEquals(repl, datanodeinfos.length);
171    final DataNode dn = cluster.getDataNode(datanodeinfos[0].getIpcPort());
172    final FSDataset data = (FSDataset)dn.getFSDataset();
173    final RandomAccessFile raf = new RandomAccessFile(data.getBlockFile(blk), "rw");
174    AppendTestUtil.LOG.info("dn=" + dn + ", blk=" + blk + " (length=" + blk.getNumBytes() + ")");
175    assertEquals(len1, raf.length());
176    raf.setLength(0);
177    raf.close();
178
179    //c. Open file in "append mode".  Append a new block worth of data. Close file.
180    final int len2 = (int)BLOCK_SIZE; 
181    {
182      FSDataOutputStream out = fs.append(p);
183      AppendTestUtil.write(out, len1, len2);
184      out.close();
185    }
186
187    //d. Reopen file and read two blocks worth of data.
188    AppendTestUtil.check(fs, p, len1 + len2);
189  }
190
191  /** TC11: Racing rename */
192  public void testTC11() throws Exception {
193    final Path p = new Path("/TC11/foo");
194    System.out.println("p=" + p);
195
196    //a. Create file and write one block of data. Close file.
197    final int len1 = (int)BLOCK_SIZE; 
198    {
199      FSDataOutputStream out = fs.create(p, false, buffersize, REPLICATION, BLOCK_SIZE);
200      AppendTestUtil.write(out, 0, len1);
201      out.close();
202    }
203
204    //b. Reopen file in "append" mode. Append half block of data.
205    FSDataOutputStream out = fs.append(p);
206    final int len2 = (int)BLOCK_SIZE/2; 
207    AppendTestUtil.write(out, len1, len2);
208   
209    //c. Rename file to file.new.
210    final Path pnew = new Path(p + ".new");
211    assertTrue(fs.rename(p, pnew));
212
213    //d. Close file handle that was opened in (b).
214    try {
215      out.close();
216      fail("close() should throw an exception");
217    } catch(Exception e) {
218      AppendTestUtil.LOG.info("GOOD!", e);
219    }
220
221    //wait for the lease recovery
222    cluster.setLeasePeriod(1000, 1000);
223    AppendTestUtil.sleep(5000);
224
225    //check block sizes
226    final long len = fs.getFileStatus(pnew).getLen();
227    final LocatedBlocks locatedblocks = fs.dfs.namenode.getBlockLocations(pnew.toString(), 0L, len);
228    final int numblock = locatedblocks.locatedBlockCount();
229    for(int i = 0; i < numblock; i++) {
230      final LocatedBlock lb = locatedblocks.get(i);
231      final Block blk = lb.getBlock();
232      final long size = lb.getBlockSize();
233      if (i < numblock - 1) {
234        assertEquals(BLOCK_SIZE, size);
235      }
236      for(DatanodeInfo datanodeinfo : lb.getLocations()) {
237        final DataNode dn = cluster.getDataNode(datanodeinfo.getIpcPort());
238        final BlockMetaDataInfo metainfo = dn.getBlockMetaDataInfo(blk);
239        assertEquals(size, metainfo.getNumBytes());
240      }
241    }
242  }
243
244  /** TC12: Append to partial CRC chunk */
245  public void testTC12() throws Exception {
246    final Path p = new Path("/TC12/foo");
247    System.out.println("p=" + p);
248   
249    //a. Create file with a block size of 64KB
250    //   and a default io.bytes.per.checksum of 512 bytes.
251    //   Write 25687 bytes of data. Close file.
252    final int len1 = 25687; 
253    {
254      FSDataOutputStream out = fs.create(p, false, buffersize, REPLICATION, BLOCK_SIZE);
255      AppendTestUtil.write(out, 0, len1);
256      out.close();
257    }
258
259    //b. Reopen file in "append" mode. Append another 5877 bytes of data. Close file.
260    final int len2 = 5877; 
261    {
262      FSDataOutputStream out = fs.append(p);
263      AppendTestUtil.write(out, len1, len2);
264      out.close();
265    }
266
267    //c. Reopen file and read 25687+5877 bytes of data from file. Close file.
268    AppendTestUtil.check(fs, p, len1 + len2);
269  }
270}
Note: See TracBrowser for help on using the repository browser.