Skip to content

Commit bb19cb4

Browse files
committed
Allow write buffer size as a parameter from hadoop/core-site.xml.
to adjust java write buffer, specify fs.glusterfs.write.buffer.size in core-site.xml in bytes.
1 parent a324ce4 commit bb19cb4

File tree

2 files changed

+18
-14
lines changed

2 files changed

+18
-14
lines changed

src/main/java/org/apache/hadoop/fs/glusterfs/GlusterFUSEOutputStream.java

Lines changed: 7 additions & 5 deletions
Original file line numberDiff line numberDiff line change
@@ -21,22 +21,24 @@
2121

2222
import java.io.*;
2323

24-
import org.apache.hadoop.fs.FSOutputSummer;
25-
import org.apache.hadoop.fs.FileSystem;
26-
2724
public class GlusterFUSEOutputStream extends OutputStream{
2825
File f;
2926
long pos;
3027
boolean closed;
3128
OutputStream fuseOutputStream;
3229

3330
public GlusterFUSEOutputStream(String file, boolean append) throws IOException{
31+
this(file,append,0);
32+
}
33+
34+
public GlusterFUSEOutputStream(String file, boolean append, int bufferSize) throws IOException{
3435
this.f=new File(file); /* not needed ? */
3536
this.pos=0;
36-
this.fuseOutputStream=new BufferedOutputStream(new FileOutputStream(file, append));
37+
this.fuseOutputStream=(bufferSize==0) ? new FileOutputStream(file, append) : new BufferedOutputStream(new FileOutputStream(file, append), bufferSize);
3738
this.closed=false;
3839
}
39-
40+
41+
4042
public long getPos() throws IOException{
4143
return pos;
4244
}

src/main/java/org/apache/hadoop/fs/glusterfs/GlusterFileSystem.java

Lines changed: 11 additions & 9 deletions
Original file line numberDiff line numberDiff line change
@@ -23,26 +23,26 @@
2323
*/
2424
package org.apache.hadoop.fs.glusterfs;
2525

26-
import java.io.*;
27-
import java.net.*;
28-
26+
import java.io.File;
27+
import java.io.FileNotFoundException;
28+
import java.io.IOException;
29+
import java.net.InetAddress;
30+
import java.net.URI;
2931
import java.util.StringTokenizer;
3032
import java.util.TreeMap;
31-
import java.util.regex.*;
3233

3334
import org.apache.hadoop.conf.Configuration;
35+
import org.apache.hadoop.fs.BlockLocation;
3436
import org.apache.hadoop.fs.FSDataInputStream;
3537
import org.apache.hadoop.fs.FSDataOutputStream;
36-
import org.apache.hadoop.fs.FileSystem;
3738
import org.apache.hadoop.fs.FileStatus;
39+
import org.apache.hadoop.fs.FileSystem;
3840
import org.apache.hadoop.fs.FileUtil;
3941
import org.apache.hadoop.fs.Path;
40-
import org.apache.hadoop.fs.BlockLocation;
4142
import org.apache.hadoop.fs.permission.FsPermission;
4243
import org.apache.hadoop.util.Progressable;
4344
import org.apache.hadoop.util.Shell;
4445
import org.apache.hadoop.util.StringUtils;
45-
4646
import org.slf4j.Logger;
4747
import org.slf4j.LoggerFactory;
4848

@@ -65,6 +65,7 @@ public class GlusterFileSystem extends FileSystem{
6565
private Path workingDir=null;
6666
private String glusterMount=null;
6767
private boolean mounted=false;
68+
private int writeBufferSize = 0;
6869

6970

7071
/* for quick IO */
@@ -111,6 +112,7 @@ public void initialize(URI uri,Configuration conf) throws IOException{
111112
String remoteGFSServer=null;
112113
String needQuickRead=null;
113114
boolean autoMount=true;
115+
114116

115117
if(this.mounted)
116118
return;
@@ -123,7 +125,7 @@ public void initialize(URI uri,Configuration conf) throws IOException{
123125
remoteGFSServer=conf.get("fs.glusterfs.server", null);
124126
needQuickRead=conf.get("quick.slave.io", null);
125127
autoMount=conf.getBoolean("fs.glusterfs.automount", true);
126-
128+
writeBufferSize = conf.getInt("fs.glusterfs.write.buffer.size", 0);
127129
/*
128130
* bail out if we do not have enough information to do a FUSE mount
129131
*/
@@ -421,7 +423,7 @@ public FSDataOutputStream create(Path path,FsPermission permission,boolean overw
421423
parent=path.getParent();
422424
mkdirs(parent);
423425

424-
glusterFileStream=new FSDataOutputStream(new GlusterFUSEOutputStream(f.getPath(), false));
426+
glusterFileStream=new FSDataOutputStream(new GlusterFUSEOutputStream(f.getPath(), false, writeBufferSize));
425427

426428
return glusterFileStream;
427429
}

0 commit comments

Comments
 (0)