|
@@ -239,26 +239,35 @@ class CopyCommands {
|
|
* Copy local files to a remote filesystem
|
|
* Copy local files to a remote filesystem
|
|
*/
|
|
*/
|
|
public static class Put extends CommandWithDestination {
|
|
public static class Put extends CommandWithDestination {
|
|
|
|
+ private ThreadPoolExecutor executor = null;
|
|
|
|
+ private int numThreads = 1;
|
|
|
|
+
|
|
|
|
+ private static final int MAX_THREADS =
|
|
|
|
+ Runtime.getRuntime().availableProcessors() * 2;
|
|
|
|
+
|
|
public static final String NAME = "put";
|
|
public static final String NAME = "put";
|
|
public static final String USAGE =
|
|
public static final String USAGE =
|
|
- "[-f] [-p] [-l] [-d] <localsrc> ... <dst>";
|
|
|
|
|
|
+ "[-f] [-p] [-l] [-d] [-t <thread count>] <localsrc> ... <dst>";
|
|
public static final String DESCRIPTION =
|
|
public static final String DESCRIPTION =
|
|
- "Copy files from the local file system " +
|
|
|
|
- "into fs. Copying fails if the file already " +
|
|
|
|
- "exists, unless the -f flag is given.\n" +
|
|
|
|
- "Flags:\n" +
|
|
|
|
- " -p : Preserves access and modification times, ownership and the mode.\n" +
|
|
|
|
- " -f : Overwrites the destination if it already exists.\n" +
|
|
|
|
- " -l : Allow DataNode to lazily persist the file to disk. Forces\n" +
|
|
|
|
- " replication factor of 1. This flag will result in reduced\n" +
|
|
|
|
- " durability. Use with care.\n" +
|
|
|
|
|
|
+ "Copy files from the local file system " +
|
|
|
|
+ "into fs. Copying fails if the file already " +
|
|
|
|
+ "exists, unless the -f flag is given.\n" +
|
|
|
|
+ "Flags:\n" +
|
|
|
|
+ " -p : Preserves timestamps, ownership and the mode.\n" +
|
|
|
|
+ " -f : Overwrites the destination if it already exists.\n" +
|
|
|
|
+ " -t <thread count> : Number of threads to be used, default is 1.\n" +
|
|
|
|
+ " -l : Allow DataNode to lazily persist the file to disk. Forces" +
|
|
|
|
+ " replication factor of 1. This flag will result in reduced" +
|
|
|
|
+ " durability. Use with care.\n" +
|
|
" -d : Skip creation of temporary file(<dst>._COPYING_).\n";
|
|
" -d : Skip creation of temporary file(<dst>._COPYING_).\n";
|
|
|
|
|
|
@Override
|
|
@Override
|
|
protected void processOptions(LinkedList<String> args) throws IOException {
|
|
protected void processOptions(LinkedList<String> args) throws IOException {
|
|
CommandFormat cf =
|
|
CommandFormat cf =
|
|
new CommandFormat(1, Integer.MAX_VALUE, "f", "p", "l", "d");
|
|
new CommandFormat(1, Integer.MAX_VALUE, "f", "p", "l", "d");
|
|
|
|
+ cf.addOptionWithValue("t");
|
|
cf.parse(args);
|
|
cf.parse(args);
|
|
|
|
+ setNumberThreads(cf.getOptValue("t"));
|
|
setOverwrite(cf.getOpt("f"));
|
|
setOverwrite(cf.getOpt("f"));
|
|
setPreserve(cf.getOpt("p"));
|
|
setPreserve(cf.getOpt("p"));
|
|
setLazyPersist(cf.getOpt("l"));
|
|
setLazyPersist(cf.getOpt("l"));
|
|
@@ -288,32 +297,22 @@ class CopyCommands {
|
|
copyStreamToTarget(System.in, getTargetPath(args.get(0)));
|
|
copyStreamToTarget(System.in, getTargetPath(args.get(0)));
|
|
return;
|
|
return;
|
|
}
|
|
}
|
|
- super.processArguments(args);
|
|
|
|
- }
|
|
|
|
- }
|
|
|
|
|
|
|
|
- public static class CopyFromLocal extends Put {
|
|
|
|
- private ThreadPoolExecutor executor = null;
|
|
|
|
- private int numThreads = 1;
|
|
|
|
|
|
+ executor = new ThreadPoolExecutor(numThreads, numThreads, 1,
|
|
|
|
+ TimeUnit.SECONDS, new ArrayBlockingQueue<>(1024),
|
|
|
|
+ new ThreadPoolExecutor.CallerRunsPolicy());
|
|
|
|
+ super.processArguments(args);
|
|
|
|
|
|
- private static final int MAX_THREADS =
|
|
|
|
- Runtime.getRuntime().availableProcessors() * 2;
|
|
|
|
- public static final String NAME = "copyFromLocal";
|
|
|
|
- public static final String USAGE =
|
|
|
|
- "[-f] [-p] [-l] [-d] [-t <thread count>] <localsrc> ... <dst>";
|
|
|
|
- public static final String DESCRIPTION =
|
|
|
|
- "Copy files from the local file system " +
|
|
|
|
- "into fs. Copying fails if the file already " +
|
|
|
|
- "exists, unless the -f flag is given.\n" +
|
|
|
|
- "Flags:\n" +
|
|
|
|
- " -p : Preserves access and modification times, ownership and the" +
|
|
|
|
- " mode.\n" +
|
|
|
|
- " -f : Overwrites the destination if it already exists.\n" +
|
|
|
|
- " -t <thread count> : Number of threads to be used, default is 1.\n" +
|
|
|
|
- " -l : Allow DataNode to lazily persist the file to disk. Forces" +
|
|
|
|
- " replication factor of 1. This flag will result in reduced" +
|
|
|
|
- " durability. Use with care.\n" +
|
|
|
|
- " -d : Skip creation of temporary file(<dst>._COPYING_).\n";
|
|
|
|
|
|
+ // issue the command and then wait for it to finish
|
|
|
|
+ executor.shutdown();
|
|
|
|
+ try {
|
|
|
|
+ executor.awaitTermination(Long.MAX_VALUE, TimeUnit.MINUTES);
|
|
|
|
+ } catch (InterruptedException e) {
|
|
|
|
+ executor.shutdownNow();
|
|
|
|
+ displayError(e);
|
|
|
|
+ Thread.currentThread().interrupt();
|
|
|
|
+ }
|
|
|
|
+ }
|
|
|
|
|
|
private void setNumberThreads(String numberThreadsString) {
|
|
private void setNumberThreads(String numberThreadsString) {
|
|
if (numberThreadsString == null) {
|
|
if (numberThreadsString == null) {
|
|
@@ -330,22 +329,6 @@ class CopyCommands {
|
|
}
|
|
}
|
|
}
|
|
}
|
|
|
|
|
|
- @Override
|
|
|
|
- protected void processOptions(LinkedList<String> args) throws IOException {
|
|
|
|
- CommandFormat cf =
|
|
|
|
- new CommandFormat(1, Integer.MAX_VALUE, "f", "p", "l", "d");
|
|
|
|
- cf.addOptionWithValue("t");
|
|
|
|
- cf.parse(args);
|
|
|
|
- setNumberThreads(cf.getOptValue("t"));
|
|
|
|
- setOverwrite(cf.getOpt("f"));
|
|
|
|
- setPreserve(cf.getOpt("p"));
|
|
|
|
- setLazyPersist(cf.getOpt("l"));
|
|
|
|
- setDirectWrite(cf.getOpt("d"));
|
|
|
|
- getRemoteDestination(args);
|
|
|
|
- // should have a -r option
|
|
|
|
- setRecursive(true);
|
|
|
|
- }
|
|
|
|
-
|
|
|
|
private void copyFile(PathData src, PathData target) throws IOException {
|
|
private void copyFile(PathData src, PathData target) throws IOException {
|
|
if (isPathRecursable(src)) {
|
|
if (isPathRecursable(src)) {
|
|
throw new PathIsDirectoryException(src.toString());
|
|
throw new PathIsDirectoryException(src.toString());
|
|
@@ -372,25 +355,6 @@ class CopyCommands {
|
|
executor.submit(task);
|
|
executor.submit(task);
|
|
}
|
|
}
|
|
|
|
|
|
- @Override
|
|
|
|
- protected void processArguments(LinkedList<PathData> args)
|
|
|
|
- throws IOException {
|
|
|
|
- executor = new ThreadPoolExecutor(numThreads, numThreads, 1,
|
|
|
|
- TimeUnit.SECONDS, new ArrayBlockingQueue<>(1024),
|
|
|
|
- new ThreadPoolExecutor.CallerRunsPolicy());
|
|
|
|
- super.processArguments(args);
|
|
|
|
-
|
|
|
|
- // issue the command and then wait for it to finish
|
|
|
|
- executor.shutdown();
|
|
|
|
- try {
|
|
|
|
- executor.awaitTermination(Long.MAX_VALUE, TimeUnit.MINUTES);
|
|
|
|
- } catch (InterruptedException e) {
|
|
|
|
- executor.shutdownNow();
|
|
|
|
- displayError(e);
|
|
|
|
- Thread.currentThread().interrupt();
|
|
|
|
- }
|
|
|
|
- }
|
|
|
|
-
|
|
|
|
@VisibleForTesting
|
|
@VisibleForTesting
|
|
public int getNumThreads() {
|
|
public int getNumThreads() {
|
|
return numThreads;
|
|
return numThreads;
|
|
@@ -401,6 +365,12 @@ class CopyCommands {
|
|
return executor;
|
|
return executor;
|
|
}
|
|
}
|
|
}
|
|
}
|
|
|
|
+
|
|
|
|
+ public static class CopyFromLocal extends Put {
|
|
|
|
+ public static final String NAME = "copyFromLocal";
|
|
|
|
+ public static final String USAGE = Put.USAGE;
|
|
|
|
+ public static final String DESCRIPTION = "Identical to the -put command.";
|
|
|
|
+ }
|
|
|
|
|
|
public static class CopyToLocal extends Get {
|
|
public static class CopyToLocal extends Get {
|
|
public static final String NAME = "copyToLocal";
|
|
public static final String NAME = "copyToLocal";
|