Browse Source

HADOOP-676. Improved exceptions and error messages for common job input specification errors. Contributed by Sanjay.

git-svn-id: https://svn.apache.org/repos/asf/lucene/hadoop/trunk@483644 13f79535-47bb-0310-9956-ffa450edef68
Doug Cutting 18 years ago
parent
commit
52bd74b1d1

+ 3 - 0
CHANGES.txt

@@ -27,6 +27,9 @@ Trunk (unreleased changes)
     default, adding a -crc option to force their creation.
     (Milind Bhandarkar via cutting)
 
+ 8. HADOOP-676. Improved exceptions and error messages for common job
+    input specification errors.  (Sanjay Dahiya via cutting)
+
 
 Release 0.9.1 - 2006-12-06
 

+ 14 - 1
src/contrib/streaming/src/java/org/apache/hadoop/streaming/StreamJob.java

@@ -19,6 +19,7 @@
 package org.apache.hadoop.streaming;
 
 import java.io.File;
+import java.io.FileNotFoundException;
 import java.io.IOException;
 import java.net.URI;
 import java.net.URISyntaxException;
@@ -37,6 +38,8 @@ import org.apache.hadoop.conf.Configuration;
 import org.apache.hadoop.io.Text;
 import org.apache.hadoop.fs.Path;
 
+import org.apache.hadoop.mapred.FileAlreadyExistsException;
+import org.apache.hadoop.mapred.InvalidJobConfException;
 import org.apache.hadoop.mapred.JobConf;
 import org.apache.hadoop.mapred.JobClient;
 import org.apache.hadoop.mapred.RunningJob;
@@ -786,7 +789,17 @@ public class StreamJob {
       LOG.info("Job complete: " + jobId_);
       LOG.info("Output: " + output_);
       error = false;
-    } finally {
+    } catch(FileNotFoundException fe){
+        LOG.error("Error launching job , bad input path : " + fe.getMessage());
+      }catch(InvalidJobConfException je){
+        LOG.error("Error launching job , Invalid job conf : " + je.getMessage());
+      }catch(FileAlreadyExistsException fae){
+        LOG.error("Error launching job , Output path already exists : " 
+            + fae.getMessage());
+      }catch( IOException ioe){
+        LOG.error("Error Launching job : " + ioe.getMessage());
+      }
+      finally {
       if (error && (running_ != null)) {
         LOG.info("killJob...");
         running_.killJob();

+ 38 - 0
src/java/org/apache/hadoop/mapred/FileAlreadyExistsException.java

@@ -0,0 +1,38 @@
+/**
+ * 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.mapred;
+
+import java.io.IOException;
+
+/**
+ * Used when target file already exists for any operation and 
+ * is not configured to be overwritten.  
+ *
+ */
+public class FileAlreadyExistsException
+    extends IOException {
+
+  public FileAlreadyExistsException() {
+    super();
+  }
+
+  public FileAlreadyExistsException(String msg) {
+    super(msg);
+  }
+}

+ 40 - 0
src/java/org/apache/hadoop/mapred/InvalidFileTypeException.java

@@ -0,0 +1,40 @@
+/**
+ * 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.mapred;
+
+import java.io.IOException;
+
+/**
+ * Used when file type differs from the desired file type. like 
+ * getting a file when a directory is expected. Or a wrong file type. 
+ * @author sanjaydahiya
+ *
+ */
+public class InvalidFileTypeException
+    extends IOException {
+
+  public InvalidFileTypeException() {
+    super();
+  }
+
+  public InvalidFileTypeException(String msg) {
+    super(msg);
+  }
+
+}

+ 39 - 0
src/java/org/apache/hadoop/mapred/InvalidJobConfException.java

@@ -0,0 +1,39 @@
+/**
+ * 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.mapred;
+
+import java.io.IOException;
+
+/**
+ * This exception is thrown when jobconf misses some mendatory attributes
+ * or value of some attributes is invalid. 
+ *
+ */
+public class InvalidJobConfException
+    extends IOException {
+
+  public InvalidJobConfException() {
+    super();
+  }
+
+  public InvalidJobConfException(String msg) {
+    super(msg);
+  }
+
+}

+ 25 - 7
src/java/org/apache/hadoop/mapred/JobClient.java

@@ -17,6 +17,7 @@
  */
 package org.apache.hadoop.mapred;
 
+import org.apache.commons.cli2.validation.InvalidArgumentException;
 import org.apache.commons.logging.*;
 
 import org.apache.hadoop.fs.*;
@@ -224,7 +225,8 @@ public class JobClient extends ToolBase implements MRConstants  {
     /**
      * Submit a job to the MR system
      */
-    public RunningJob submitJob(String jobFile) throws IOException {
+    public RunningJob submitJob(String jobFile) throws FileNotFoundException, 
+      InvalidJobConfException,IOException {
         // Load in the submitted job details
         JobConf job = new JobConf(jobFile);
         return submitJob(job);
@@ -234,7 +236,8 @@ public class JobClient extends ToolBase implements MRConstants  {
     /**
      * Submit a job to the MR system
      */
-    public RunningJob submitJob(JobConf job) throws IOException {
+    public RunningJob submitJob(JobConf job) throws FileNotFoundException, 
+      InvalidJobConfException, IOException {
         //
         // First figure out what fs the JobTracker is using.  Copy the
         // job to it, under a temporary name.  This allows DFS to work,
@@ -301,17 +304,33 @@ public class JobClient extends ToolBase implements MRConstants  {
 
         FileSystem userFileSys = FileSystem.get(job);
         Path[] inputDirs = job.getInputPaths();
+        // input paths should exist. 
+        
         boolean[] validDirs = 
           job.getInputFormat().areValidInputDirectories(userFileSys, inputDirs);
         for(int i=0; i < validDirs.length; ++i) {
           if (!validDirs[i]) {
-            String msg = "Input directory " + inputDirs[i] + 
-                         " in " + userFileSys.getName() + " is invalid.";
-            LOG.error(msg);
-            throw new IOException(msg);
+            String msg = null ; 
+            if( !userFileSys.exists(inputDirs[i]) ){
+              msg = "Input directory " + inputDirs[i] + 
+                         " doesn't exist in " + userFileSys.getName();
+              LOG.error(msg);
+              throw new FileNotFoundException(msg);
+            }else if( !userFileSys.isDirectory(inputDirs[i])){
+              msg = "Invalid input path, expecting directory : " + inputDirs[i] ;
+              LOG.error(msg); 
+              throw new InvalidFileTypeException(msg);  
+            }else{
+              // some other error
+              msg = "Input directory " + inputDirs[i] + 
+                           " in " + userFileSys.getName() + " is invalid.";
+              LOG.error(msg);
+              throw new IOException(msg);
+            }
           }
         }
 
+
         // Check the output specification
         job.getOutputFormat().checkOutputSpecs(fs, job);
 
@@ -428,7 +447,6 @@ public class JobClient extends ToolBase implements MRConstants  {
         
 
     public int run(String[] argv) throws Exception {
-        // TODO Auto-generated method stub
         if (argv.length < 2) {
             System.out.println("JobClient -submit <job> | -status <id> | -kill <id> [-jt <jobtracker:port>|<config>]");
             System.exit(-1);

+ 6 - 4
src/java/org/apache/hadoop/mapred/OutputFormatBase.java

@@ -84,15 +84,17 @@ public abstract class OutputFormatBase implements OutputFormat {
                                                Progressable progress)
     throws IOException;
 
-  public void checkOutputSpecs(FileSystem fs, JobConf job) throws IOException {
+  public void checkOutputSpecs(FileSystem fs, JobConf job) 
+          throws FileAlreadyExistsException, 
+             InvalidJobConfException, IOException {
     // Ensure that the output directory is set and not already there
     Path outDir = job.getOutputPath();
     if (outDir == null && job.getNumReduceTasks() != 0) {
-      throw new IOException("Output directory not set in JobConf.");
+      throw new InvalidJobConfException("Output directory not set in JobConf.");
     }
     if (outDir != null && fs.exists(outDir)) {
-      throw new IOException("Output directory " + outDir + 
-                            " already exists.");
+      throw new FileAlreadyExistsException("Output directory " + outDir + 
+                            " already exists in " + fs.getName() );
     }
   }