Author: dhruba
Date: Wed Jun 4 10:45:50 2008
New Revision: 663326
URL: http://svn.apache.org/viewvc?rev=663326&view=rev
Log:
HADOOP-3177. Implement Syncable interface for FileSystem.
(Tsz Wo (Nicholas), SZE via dhruba)
Added:
hadoop/core/trunk/src/java/org/apache/hadoop/fs/Syncable.java
Modified:
hadoop/core/trunk/CHANGES.txt
hadoop/core/trunk/src/java/org/apache/hadoop/dfs/DFSClient.java
hadoop/core/trunk/src/java/org/apache/hadoop/fs/FSDataOutputStream.java
hadoop/core/trunk/src/java/org/apache/hadoop/fs/RawLocalFileSystem.java
hadoop/core/trunk/src/test/org/apache/hadoop/dfs/TestAbandonBlock.java
hadoop/core/trunk/src/test/org/apache/hadoop/dfs/TestFileAppend.java
hadoop/core/trunk/src/test/org/apache/hadoop/dfs/TestFileCreation.java
Modified: hadoop/core/trunk/CHANGES.txt
URL:
http://svn.apache.org/viewvc/hadoop/core/trunk/CHANGES.txt?rev=663326&r1=663325&r2=663326&view=diff
==============================================================================
--- hadoop/core/trunk/CHANGES.txt (original)
+++ hadoop/core/trunk/CHANGES.txt Wed Jun 4 10:45:50 2008
@@ -126,6 +126,9 @@
HADOOP-3250. Extend FileSystem API to allow appending to files.
(Tsz Wo (Nicholas), SZE via cdouglas)
+ HADOOP-3177. Implement Syncable interface for FileSystem.
+ (Tsz Wo (Nicholas), SZE via dhruba)
+
IMPROVEMENTS
HADOOP-2928. Remove deprecated FileSystem.getContentLength().
Modified: hadoop/core/trunk/src/java/org/apache/hadoop/dfs/DFSClient.java
URL:
http://svn.apache.org/viewvc/hadoop/core/trunk/src/java/org/apache/hadoop/dfs/DFSClient.java?rev=663326&r1=663325&r2=663326&view=diff
==============================================================================
--- hadoop/core/trunk/src/java/org/apache/hadoop/dfs/DFSClient.java (original)
+++ hadoop/core/trunk/src/java/org/apache/hadoop/dfs/DFSClient.java Wed Jun 4
10:45:50 2008
@@ -1699,7 +1699,7 @@
* datanode from the original pipeline. The DataStreamer now
* starts sending packets from the dataQueue.
****************************************************************/
- class DFSOutputStream extends FSOutputSummer {
+ class DFSOutputStream extends FSOutputSummer implements Syncable {
private Socket s;
boolean closed = false;
@@ -2497,7 +2497,7 @@
* that data has been flushed to persistent store on the
* datanode. Block allocations are persisted on namenode.
*/
- public synchronized void fsync() throws IOException {
+ public synchronized void sync() throws IOException {
try {
/* Record current blockOffset. This might be changed inside
* flushBuffer() where a partial checksum chunk might be flushed.
Modified:
hadoop/core/trunk/src/java/org/apache/hadoop/fs/FSDataOutputStream.java
URL:
http://svn.apache.org/viewvc/hadoop/core/trunk/src/java/org/apache/hadoop/fs/FSDataOutputStream.java?rev=663326&r1=663325&r2=663326&view=diff
==============================================================================
--- hadoop/core/trunk/src/java/org/apache/hadoop/fs/FSDataOutputStream.java
(original)
+++ hadoop/core/trunk/src/java/org/apache/hadoop/fs/FSDataOutputStream.java Wed
Jun 4 10:45:50 2008
@@ -22,7 +22,7 @@
/** Utility that wraps a [EMAIL PROTECTED] OutputStream} in a [EMAIL
PROTECTED] DataOutputStream},
* buffers output through a [EMAIL PROTECTED] BufferedOutputStream} and
creates a checksum
* file. */
-public class FSDataOutputStream extends DataOutputStream {
+public class FSDataOutputStream extends DataOutputStream implements Syncable {
private OutputStream wrappedStream;
private static class PositionCache extends FilterOutputStream {
@@ -83,4 +83,11 @@
public OutputStream getWrappedStream() {
return wrappedStream;
}
+
+ /** [EMAIL PROTECTED] */
+ public void sync() throws IOException {
+ if (wrappedStream instanceof Syncable) {
+ ((Syncable)wrappedStream).sync();
+ }
+ }
}
Modified:
hadoop/core/trunk/src/java/org/apache/hadoop/fs/RawLocalFileSystem.java
URL:
http://svn.apache.org/viewvc/hadoop/core/trunk/src/java/org/apache/hadoop/fs/RawLocalFileSystem.java?rev=663326&r1=663325&r2=663326&view=diff
==============================================================================
--- hadoop/core/trunk/src/java/org/apache/hadoop/fs/RawLocalFileSystem.java
(original)
+++ hadoop/core/trunk/src/java/org/apache/hadoop/fs/RawLocalFileSystem.java Wed
Jun 4 10:45:50 2008
@@ -180,7 +180,7 @@
/*********************************************************
* For create()'s FSOutputStream.
*********************************************************/
- class LocalFSFileOutputStream extends OutputStream {
+ class LocalFSFileOutputStream extends OutputStream implements Syncable {
FileOutputStream fos;
private LocalFSFileOutputStream(Path f, boolean append) throws IOException
{
@@ -207,6 +207,11 @@
throw new FSError(e); // assume native fs error
}
}
+
+ /** [EMAIL PROTECTED] */
+ public void sync() throws IOException {
+ fos.getFD().sync();
+ }
}
/** [EMAIL PROTECTED] */
Added: hadoop/core/trunk/src/java/org/apache/hadoop/fs/Syncable.java
URL:
http://svn.apache.org/viewvc/hadoop/core/trunk/src/java/org/apache/hadoop/fs/Syncable.java?rev=663326&view=auto
==============================================================================
--- hadoop/core/trunk/src/java/org/apache/hadoop/fs/Syncable.java (added)
+++ hadoop/core/trunk/src/java/org/apache/hadoop/fs/Syncable.java Wed Jun 4
10:45:50 2008
@@ -0,0 +1,30 @@
+/**
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements. See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership. The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License. You may obtain a copy of the License at
+ *
+ * http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS,
+ * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ * See the License for the specific language governing permissions and
+ * limitations under the License.
+ */
+
+package org.apache.hadoop.fs;
+
+import java.io.IOException;
+
+/** This interface declare the sync() operation. */
+public interface Syncable {
+ /**
+ * Synchronize all buffer with the underlying devices.
+ * @throws IOException
+ */
+ public void sync() throws IOException;
+}
\ No newline at end of file
Modified: hadoop/core/trunk/src/test/org/apache/hadoop/dfs/TestAbandonBlock.java
URL:
http://svn.apache.org/viewvc/hadoop/core/trunk/src/test/org/apache/hadoop/dfs/TestAbandonBlock.java?rev=663326&r1=663325&r2=663326&view=diff
==============================================================================
--- hadoop/core/trunk/src/test/org/apache/hadoop/dfs/TestAbandonBlock.java
(original)
+++ hadoop/core/trunk/src/test/org/apache/hadoop/dfs/TestAbandonBlock.java Wed
Jun 4 10:45:50 2008
@@ -24,7 +24,6 @@
import org.apache.hadoop.conf.Configuration;
import org.apache.hadoop.fs.*;
import org.apache.hadoop.util.StringUtils;
-import org.apache.hadoop.net.NetUtils;
public class TestAbandonBlock extends junit.framework.TestCase {
public static final Log LOG = LogFactory.getLog(TestAbandonBlock.class);
@@ -33,12 +32,6 @@
static final String FILE_NAME_PREFIX
= "/" + TestAbandonBlock.class.getSimpleName() + "_";
- private void flushFile(FSDataOutputStream stm) throws IOException {
- DFSClient.DFSOutputStream dfstream = (DFSClient.DFSOutputStream)
- (stm.getWrappedStream());
- dfstream.fsync();
- }
-
public void testAbandonBlock() throws IOException {
MiniDFSCluster cluster = new MiniDFSCluster(CONF, 2, true, null);
FileSystem fs = cluster.getFileSystem();
@@ -51,7 +44,7 @@
for(int i = 0; i < 1024; i++) {
fout.write(123);
}
- flushFile(fout);
+ fout.sync();
//try reading the block by someone
DFSClient dfsclient = new DFSClient(CONF);
Modified: hadoop/core/trunk/src/test/org/apache/hadoop/dfs/TestFileAppend.java
URL:
http://svn.apache.org/viewvc/hadoop/core/trunk/src/test/org/apache/hadoop/dfs/TestFileAppend.java?rev=663326&r1=663325&r2=663326&view=diff
==============================================================================
--- hadoop/core/trunk/src/test/org/apache/hadoop/dfs/TestFileAppend.java
(original)
+++ hadoop/core/trunk/src/test/org/apache/hadoop/dfs/TestFileAppend.java Wed
Jun 4 10:45:50 2008
@@ -63,12 +63,6 @@
return stm;
}
- private void flushFile(FSDataOutputStream stm) throws IOException {
- DFSClient.DFSOutputStream dfstream = (DFSClient.DFSOutputStream)
- (stm.getWrappedStream());
- dfstream.fsync();
- }
-
//
// writes to file but does not close it
//
@@ -234,14 +228,14 @@
// write to file
int mid = fileSize/2;
stm.write(fileContents, 0, mid);
- flushFile(stm);
+ stm.sync();
System.out.println("Wrote and Flushed first part of file.");
// write the remainder of the file
stm.write(fileContents, mid, fileSize - mid);
System.out.println("Written second part of file");
- flushFile(stm);
- flushFile(stm); // two consecutive flushes is being tested here.
+ stm.sync();
+ stm.sync();
System.out.println("Wrote and Flushed second part of file.");
// verify that full blocks are sane
@@ -287,7 +281,7 @@
int start = 0;
for (start = 0; (start + 29) < fileSize; ) {
stm.write(fileContents, start, 29);
- flushFile(stm);
+ stm.sync();
start += 29;
}
stm.write(fileContents, start, fileSize-start);
Modified: hadoop/core/trunk/src/test/org/apache/hadoop/dfs/TestFileCreation.java
URL:
http://svn.apache.org/viewvc/hadoop/core/trunk/src/test/org/apache/hadoop/dfs/TestFileCreation.java?rev=663326&r1=663325&r2=663326&view=diff
==============================================================================
--- hadoop/core/trunk/src/test/org/apache/hadoop/dfs/TestFileCreation.java
(original)
+++ hadoop/core/trunk/src/test/org/apache/hadoop/dfs/TestFileCreation.java Wed
Jun 4 10:45:50 2008
@@ -62,12 +62,6 @@
return stm;
}
- private void flushFile(FSDataOutputStream stm) throws IOException {
- DFSClient.DFSOutputStream dfstream = (DFSClient.DFSOutputStream)
- (stm.getWrappedStream());
- dfstream.fsync();
- }
-
//
// writes to file but does not close it
//
@@ -439,7 +433,7 @@
// write two full blocks.
writeFile(stm, numBlocks * blockSize);
- flushFile(stm);
+ stm.sync();
// rename file wile keeping it open.
Path fileRenamed = new Path("/filestatusRenamed.dat");
@@ -657,7 +651,7 @@
final Path fpath = new Path(f);
FSDataOutputStream out = TestFileCreation.createFile(dfs, fpath,
DATANODE_NUM);
out.write("something".getBytes());
- ((DFSClient.DFSOutputStream)out.getWrappedStream()).fsync();
+ out.sync();
// set the soft and hard limit to be 1 second so that the
// namenode triggers lease recovery