You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@airavata.apache.org by go...@apache.org on 2017/05/10 15:54:26 UTC

airavata git commit: Add OutputParser implementations for JobSubmissionTask

Repository: airavata
Updated Branches:
  refs/heads/feature-workload-mgmt 6a6221065 -> cc1ca3d59


Add OutputParser implementations for JobSubmissionTask


Project: http://git-wip-us.apache.org/repos/asf/airavata/repo
Commit: http://git-wip-us.apache.org/repos/asf/airavata/commit/cc1ca3d5
Tree: http://git-wip-us.apache.org/repos/asf/airavata/tree/cc1ca3d5
Diff: http://git-wip-us.apache.org/repos/asf/airavata/diff/cc1ca3d5

Branch: refs/heads/feature-workload-mgmt
Commit: cc1ca3d592bcbd87a8d8dd1ddbf72b48310e59e7
Parents: 6a62210
Author: Gourav Shenoy <go...@apache.org>
Authored: Wed May 10 11:54:24 2017 -0400
Committer: Gourav Shenoy <go...@apache.org>
Committed: Wed May 10 11:54:24 2017 -0400

----------------------------------------------------------------------
 .../AiravataCustomCommandOutputParser.java      |  59 ++++++++
 .../jobsubmission/parser/ForkOutputParser.java  |  61 ++++++++
 .../jobsubmission/parser/LSFOutputParser.java   | 130 +++++++++++++++++
 .../jobsubmission/parser/PBSOutputParser.java   | 144 +++++++++++++++++++
 .../jobsubmission/parser/SlurmOutputParser.java | 139 ++++++++++++++++++
 .../jobsubmission/parser/UGEOutputParser.java   | 111 ++++++++++++++
 .../jobsubmission/utils/JobSubmissionUtils.java |  27 ++++
 7 files changed, 671 insertions(+)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/airavata/blob/cc1ca3d5/modules/worker/task-jobsubmission/src/main/java/org/apache/airavata/worker/task/jobsubmission/parser/AiravataCustomCommandOutputParser.java
----------------------------------------------------------------------
diff --git a/modules/worker/task-jobsubmission/src/main/java/org/apache/airavata/worker/task/jobsubmission/parser/AiravataCustomCommandOutputParser.java b/modules/worker/task-jobsubmission/src/main/java/org/apache/airavata/worker/task/jobsubmission/parser/AiravataCustomCommandOutputParser.java
new file mode 100644
index 0000000..432fd76
--- /dev/null
+++ b/modules/worker/task-jobsubmission/src/main/java/org/apache/airavata/worker/task/jobsubmission/parser/AiravataCustomCommandOutputParser.java
@@ -0,0 +1,59 @@
+/*
+ *
+ * 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.airavata.worker.task.jobsubmission.parser;
+
+import org.apache.airavata.model.status.JobStatus;
+import org.apache.airavata.worker.core.cluster.OutputParser;
+import org.apache.airavata.worker.core.exceptions.WorkerException;
+import org.slf4j.Logger;
+import org.slf4j.LoggerFactory;
+
+import java.util.Map;
+
+public class AiravataCustomCommandOutputParser implements OutputParser {
+    private static final Logger log = LoggerFactory.getLogger(AiravataCustomCommandOutputParser.class);
+
+    @Override
+    public String parseJobSubmission(String rawOutput) throws WorkerException {
+        throw new UnsupportedOperationException();
+    }
+
+    @Override
+    public boolean isJobSubmissionFailed(String rawOutput) {
+        throw new UnsupportedOperationException();
+    }
+
+    @Override
+    public JobStatus parseJobStatus(String jobID, String rawOutput) throws WorkerException {
+        throw new UnsupportedOperationException();
+    }
+
+    @Override
+    public void parseJobStatuses(String userName, Map<String, JobStatus> statusMap, String rawOutput) throws WorkerException {
+        throw new UnsupportedOperationException();
+    }
+
+    @Override
+    public String parseJobId(String jobName, String rawOutput) throws WorkerException {
+        throw new UnsupportedOperationException();
+    }
+}

http://git-wip-us.apache.org/repos/asf/airavata/blob/cc1ca3d5/modules/worker/task-jobsubmission/src/main/java/org/apache/airavata/worker/task/jobsubmission/parser/ForkOutputParser.java
----------------------------------------------------------------------
diff --git a/modules/worker/task-jobsubmission/src/main/java/org/apache/airavata/worker/task/jobsubmission/parser/ForkOutputParser.java b/modules/worker/task-jobsubmission/src/main/java/org/apache/airavata/worker/task/jobsubmission/parser/ForkOutputParser.java
new file mode 100644
index 0000000..9641e40
--- /dev/null
+++ b/modules/worker/task-jobsubmission/src/main/java/org/apache/airavata/worker/task/jobsubmission/parser/ForkOutputParser.java
@@ -0,0 +1,61 @@
+/*
+ *
+ * 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.airavata.worker.task.jobsubmission.parser;
+
+import org.apache.airavata.common.utils.AiravataUtils;
+import org.apache.airavata.model.status.JobStatus;
+import org.apache.airavata.worker.core.cluster.OutputParser;
+import org.apache.airavata.worker.core.exceptions.WorkerException;
+import org.slf4j.Logger;
+import org.slf4j.LoggerFactory;
+
+import java.util.Map;
+
+public class ForkOutputParser implements OutputParser {
+    private static final Logger log = LoggerFactory.getLogger(ForkOutputParser.class);
+
+    @Override
+    public String parseJobSubmission(String rawOutput) throws WorkerException {
+	    return AiravataUtils.getId("JOB_ID_");
+    }
+
+    @Override
+    public boolean isJobSubmissionFailed(String rawOutput) {
+        return false;
+    }
+
+    @Override
+    public JobStatus parseJobStatus(String jobID, String rawOutput) throws WorkerException {
+        return null;
+    }
+
+    @Override
+    public void parseJobStatuses(String userName, Map<String, JobStatus> statusMap, String rawOutput) throws WorkerException {
+
+    }
+
+    @Override
+    public String parseJobId(String jobName, String rawOutput) throws WorkerException {
+        // For fork jobs there is no job ID, hence airavata generates a job ID
+        return AiravataUtils.getId(jobName);
+    }
+}

http://git-wip-us.apache.org/repos/asf/airavata/blob/cc1ca3d5/modules/worker/task-jobsubmission/src/main/java/org/apache/airavata/worker/task/jobsubmission/parser/LSFOutputParser.java
----------------------------------------------------------------------
diff --git a/modules/worker/task-jobsubmission/src/main/java/org/apache/airavata/worker/task/jobsubmission/parser/LSFOutputParser.java b/modules/worker/task-jobsubmission/src/main/java/org/apache/airavata/worker/task/jobsubmission/parser/LSFOutputParser.java
new file mode 100644
index 0000000..04958a9
--- /dev/null
+++ b/modules/worker/task-jobsubmission/src/main/java/org/apache/airavata/worker/task/jobsubmission/parser/LSFOutputParser.java
@@ -0,0 +1,130 @@
+/*
+ *
+ * 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.airavata.worker.task.jobsubmission.parser;
+
+import org.apache.airavata.model.status.JobState;
+import org.apache.airavata.model.status.JobStatus;
+import org.apache.airavata.worker.core.cluster.OutputParser;
+import org.apache.airavata.worker.core.exceptions.WorkerException;
+import org.slf4j.Logger;
+import org.slf4j.LoggerFactory;
+
+import java.util.ArrayList;
+import java.util.HashMap;
+import java.util.List;
+import java.util.Map;
+import java.util.regex.Matcher;
+import java.util.regex.Pattern;
+
+public class LSFOutputParser implements OutputParser {
+    private final static Logger logger = LoggerFactory.getLogger(LSFOutputParser.class);
+
+    @Override
+    public String parseJobSubmission(String rawOutput) throws WorkerException {
+        logger.debug(rawOutput);
+        return rawOutput.substring(rawOutput.indexOf("<")+1,rawOutput.indexOf(">"));
+    }
+
+    @Override
+    public boolean isJobSubmissionFailed(String rawOutput) {
+        return false;
+    }
+
+    @Override
+    public JobStatus parseJobStatus(String jobID, String rawOutput) throws WorkerException {
+        boolean jobFount = false;
+        logger.debug(rawOutput);
+        //todo this is not used anymore
+        return null;
+    }
+
+    @Override
+    public void parseJobStatuses(String userName, Map<String, JobStatus> statusMap, String rawOutput) throws WorkerException {
+        logger.debug(rawOutput);
+
+        String[]    info = rawOutput.split("\n");
+//        int lastStop = 0;
+        for (String jobID : statusMap.keySet()) {
+            String jobName = jobID.split(",")[1];
+            boolean found = false;
+            for (int i = 0; i < info.length; i++) {
+                if (info[i].contains(jobName.substring(0,8))) {
+                    // now starts processing this line
+                    logger.info(info[i]);
+                    String correctLine = info[i];
+                    String[] columns = correctLine.split(" ");
+                    List<String> columnList = new ArrayList<String>();
+                    for (String s : columns) {
+                        if (!"".equals(s)) {
+                            columnList.add(s);
+                        }
+                    }
+//                    lastStop = i + 1;
+                    try {
+	                    statusMap.put(jobID, new JobStatus(JobState.valueOf(columnList.get(2))));
+                    }catch(IndexOutOfBoundsException e) {
+	                    statusMap.put(jobID, new JobStatus(JobState.valueOf("U")));
+                    }
+                    found = true;
+                    break;
+                }
+            }
+            if(!found)
+                logger.error("Couldn't find the status of the Job with JobName: " + jobName + "Job Id: " + jobID.split(",")[0]);
+        }
+    }
+
+    @Override
+    public String parseJobId(String jobName, String rawOutput) throws WorkerException {
+        String regJobId = "jobId";
+        Pattern pattern = Pattern.compile("(?=(?<" + regJobId + ">\\d+)\\s+\\w+\\s+" + jobName + ")"); // regex - look ahead and match
+        if (rawOutput != null) {
+            Matcher matcher = pattern.matcher(rawOutput);
+            if (matcher.find()) {
+                return matcher.group(regJobId);
+            } else {
+                logger.error("No match is found for JobName");
+                return null;
+            }
+        } else {
+            logger.error("Error: RawOutput shouldn't be null");
+            return null;
+        }
+    }
+
+    public static void main(String[] args) {
+        String test = "Job <2477982> is submitted to queue <short>.";
+        System.out.println(test.substring(test.indexOf("<")+1, test.indexOf(">")));
+        String test1 = "JOBID   USER    STAT  QUEUE      FROM_HOST   EXEC_HOST   JOB_NAME   SUBMIT_TIME\n" +
+                "2636607 lg11w   RUN   long       ghpcc06     c11b02      *069656647 Mar  7 00:58\n" +
+                "2636582 lg11w   RUN   long       ghpcc06     c02b01      2134490944 Mar  7 00:48";
+        Map<String, JobStatus> statusMap = new HashMap<String, JobStatus>();
+        statusMap.put("2477983,2134490944", new JobStatus(JobState.UNKNOWN));
+        LSFOutputParser lsfOutputParser = new LSFOutputParser();
+        try {
+            lsfOutputParser.parseJobStatuses("cjh", statusMap, test1);
+        } catch (WorkerException e) {
+            logger.error(e.getMessage(), e);
+        }
+        System.out.println(statusMap.get("2477983,2134490944"));
+
+    }
+}

http://git-wip-us.apache.org/repos/asf/airavata/blob/cc1ca3d5/modules/worker/task-jobsubmission/src/main/java/org/apache/airavata/worker/task/jobsubmission/parser/PBSOutputParser.java
----------------------------------------------------------------------
diff --git a/modules/worker/task-jobsubmission/src/main/java/org/apache/airavata/worker/task/jobsubmission/parser/PBSOutputParser.java b/modules/worker/task-jobsubmission/src/main/java/org/apache/airavata/worker/task/jobsubmission/parser/PBSOutputParser.java
new file mode 100644
index 0000000..022984f
--- /dev/null
+++ b/modules/worker/task-jobsubmission/src/main/java/org/apache/airavata/worker/task/jobsubmission/parser/PBSOutputParser.java
@@ -0,0 +1,144 @@
+/*
+ *
+ * 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.airavata.worker.task.jobsubmission.parser;
+
+import org.apache.airavata.model.status.JobStatus;
+import org.apache.airavata.worker.core.cluster.OutputParser;
+import org.apache.airavata.worker.core.exceptions.WorkerException;
+import org.apache.airavata.worker.task.jobsubmission.utils.JobSubmissionUtils;
+import org.slf4j.Logger;
+import org.slf4j.LoggerFactory;
+
+import java.util.ArrayList;
+import java.util.List;
+import java.util.Map;
+import java.util.regex.Matcher;
+import java.util.regex.Pattern;
+
+public class PBSOutputParser implements OutputParser {
+    private static final Logger log = LoggerFactory.getLogger(PBSOutputParser.class);
+
+    public String parseJobSubmission(String rawOutput) {
+        log.debug(rawOutput);
+        String jobId = rawOutput;
+        if (!rawOutput.isEmpty() && rawOutput.contains("\n")){
+            String[] split = rawOutput.split("\n");
+            if (split.length != 0){
+                jobId = split[0];
+            }
+        }
+        return jobId;  //In PBS stdout is going to be directly the jobID
+    }
+
+    @Override
+    public boolean isJobSubmissionFailed(String rawOutput) {
+        return false;
+    }
+
+    public JobStatus parseJobStatus(String jobID, String rawOutput) {
+        boolean jobFount = false;
+        log.debug(rawOutput);
+        String[] info = rawOutput.split("\n");
+        String[] line = null;
+        int index = 0;
+        for (String anInfo : info) {
+            index++;
+            if (anInfo.contains("Job Id:")) {
+                if (anInfo.contains(jobID)) {
+                    jobFount = true;
+                    break;
+                }
+            }
+        }
+        if (jobFount) {
+            for (int i=index;i<info.length;i++) {
+                String anInfo = info[i];
+                if (anInfo.contains("=")) {
+                    line = anInfo.split("=", 2);
+                    if (line.length != 0) {
+                        if (line[0].contains("job_state")) {
+	                        return new JobStatus(JobSubmissionUtils.getJobState(line[1].replaceAll(" ", "")));
+                        }
+                    }
+                }
+            }
+        }
+        return null;
+    }
+
+    public void parseJobStatuses(String userName, Map<String, JobStatus> statusMap, String rawOutput) {
+        log.debug(rawOutput);
+        String[]    info = rawOutput.split("\n");
+//        int lastStop = 0;
+        for (String jobID : statusMap.keySet()) {
+            String jobName = jobID.split(",")[1];
+            boolean found = false;
+            for (int i = 0; i < info.length; i++) {
+                if (info[i].contains(jobName.substring(0,8))) {
+                    // now starts processing this line
+                    log.info(info[i]);
+                    String correctLine = info[i];
+                    String[] columns = correctLine.split(" ");
+                    List<String> columnList = new ArrayList<String>();
+                    for (String s : columns) {
+                        if (!"".equals(s)) {
+                            columnList.add(s);
+                        }
+                    }
+//                    lastStop = i + 1;
+                    try {
+	                    statusMap.put(jobID, new JobStatus(JobSubmissionUtils.getJobState(columnList.get(9))));
+                    }catch(IndexOutOfBoundsException e) {
+	                    statusMap.put(jobID, new JobStatus(JobSubmissionUtils.getJobState("U")));
+                    }
+                    found = true;
+                    break;
+                }
+            }
+            if(!found)
+            log.error("Couldn't find the status of the Job with JobName: " + jobName + "Job Id: " + jobID.split(",")[0]);
+        }
+    }
+
+    @Override
+    public String parseJobId(String jobName, String rawOutput) throws WorkerException {
+        /* output will look like
+        Job Id: 2080802.gordon-fe2.local
+            Job_Name = A312402627
+        */
+        String regJobId = "jobId";
+        Pattern pattern = Pattern.compile("(?<" + regJobId + ">[^\\s]*)\\s*.* " + jobName);
+        if (rawOutput != null) {
+            Matcher matcher = pattern.matcher(rawOutput);
+            if (matcher.find()) {
+                return matcher.group(regJobId);
+            } else {
+                log.error("No match is found for JobName");
+                return null;
+            }
+        } else {
+            log.error("Error: RawOutput shouldn't be null");
+            return null;
+        }
+    }
+
+
+}

http://git-wip-us.apache.org/repos/asf/airavata/blob/cc1ca3d5/modules/worker/task-jobsubmission/src/main/java/org/apache/airavata/worker/task/jobsubmission/parser/SlurmOutputParser.java
----------------------------------------------------------------------
diff --git a/modules/worker/task-jobsubmission/src/main/java/org/apache/airavata/worker/task/jobsubmission/parser/SlurmOutputParser.java b/modules/worker/task-jobsubmission/src/main/java/org/apache/airavata/worker/task/jobsubmission/parser/SlurmOutputParser.java
new file mode 100644
index 0000000..016a9b3
--- /dev/null
+++ b/modules/worker/task-jobsubmission/src/main/java/org/apache/airavata/worker/task/jobsubmission/parser/SlurmOutputParser.java
@@ -0,0 +1,139 @@
+/*
+ *
+ * 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.airavata.worker.task.jobsubmission.parser;
+
+import org.apache.airavata.model.status.JobState;
+import org.apache.airavata.model.status.JobStatus;
+import org.apache.airavata.worker.core.cluster.OutputParser;
+import org.apache.airavata.worker.core.exceptions.WorkerException;
+import org.apache.airavata.worker.task.jobsubmission.utils.JobSubmissionUtils;
+import org.slf4j.Logger;
+import org.slf4j.LoggerFactory;
+
+import java.util.ArrayList;
+import java.util.List;
+import java.util.Map;
+import java.util.regex.Matcher;
+import java.util.regex.Pattern;
+
+public class SlurmOutputParser implements OutputParser {
+    private static final Logger log = LoggerFactory.getLogger(SlurmOutputParser.class);
+    public static final int JOB_NAME_OUTPUT_LENGTH = 8;
+    public static final String STATUS = "status";
+	public static final String JOBID = "jobId";
+
+
+    /**
+     * This can be used to parseSingleJob the outpu of sbatch and extrac the jobID from the content
+     *
+     * @param rawOutput
+     * @return
+     */
+    public String parseJobSubmission(String rawOutput) throws WorkerException {
+	    log.info(rawOutput);
+	    Pattern pattern = Pattern.compile("Submitted batch job (?<" + JOBID + ">[^\\s]*)");
+	    Matcher matcher = pattern.matcher(rawOutput);
+	    if (matcher.find()) {
+		    return matcher.group(JOBID);
+	    }
+	    return "";
+    }
+
+    @Override
+    public boolean isJobSubmissionFailed(String rawOutput) {
+        Pattern pattern = Pattern.compile("FAILED");
+        Matcher matcher = pattern.matcher(rawOutput);
+        return matcher.find();
+    }
+
+    public JobStatus parseJobStatus(String jobID, String rawOutput) throws WorkerException {
+        log.info(rawOutput);
+        Pattern pattern = Pattern.compile(jobID + "(?=\\s+\\S+\\s+\\S+\\s+\\S+\\s+(?<" + STATUS + ">\\w+))");
+        Matcher matcher = pattern.matcher(rawOutput);
+        if (matcher.find()) {
+	        return new JobStatus(JobSubmissionUtils.getJobState(matcher.group(STATUS)));
+        }
+        return null;
+    }
+
+    public void parseJobStatuses(String userName, Map<String, JobStatus> statusMap, String rawOutput) throws WorkerException {
+        log.debug(rawOutput);
+        String[] info = rawOutput.split("\n");
+        String lastString = info[info.length - 1];
+        if (lastString.contains("JOBID") || lastString.contains("PARTITION")) {
+            log.info("There are no jobs with this username ... ");
+            return;
+        }
+//        int lastStop = 0;
+        for (String jobID : statusMap.keySet()) {
+            String jobId = jobID.split(",")[0];
+            String jobName = jobID.split(",")[1];
+            boolean found = false;
+            for (int i = 0; i < info.length; i++) {
+                if (info[i].contains(jobName.substring(0, 8))) {
+                    // now starts processing this line
+                    log.info(info[i]);
+                    String correctLine = info[i];
+                    String[] columns = correctLine.split(" ");
+                    List<String> columnList = new ArrayList<String>();
+                    for (String s : columns) {
+                        if (!"".equals(s)) {
+                            columnList.add(s);
+                        }
+                    }
+                    try {
+	                    statusMap.put(jobID, new JobStatus(JobState.valueOf(columnList.get(4))));
+                    } catch (IndexOutOfBoundsException e) {
+	                    statusMap.put(jobID, new JobStatus(JobState.valueOf("U")));
+                    }
+                    found = true;
+                    break;
+                }
+            }
+            if (!found) {
+                log.error("Couldn't find the status of the Job with JobName: " + jobName + "Job Id: " + jobId);
+            }
+        }
+    }
+
+    @Override
+    public String parseJobId(String jobName, String rawOutput) throws WorkerException {
+        String regJobId = "jobId";
+        if (jobName == null) {
+            return null;
+        } else if(jobName.length() > JOB_NAME_OUTPUT_LENGTH) {
+            jobName = jobName.substring(0, JOB_NAME_OUTPUT_LENGTH);
+        }
+        Pattern pattern = Pattern.compile("(?=(?<" + regJobId + ">\\d+)\\s+\\w+\\s+" + jobName + ")"); // regex - look ahead and match
+        if (rawOutput != null) {
+            Matcher matcher = pattern.matcher(rawOutput);
+            if (matcher.find()) {
+                return matcher.group(regJobId);
+            } else {
+                log.error("No match is found for JobName");
+                return null;
+            }
+        } else {
+            log.error("Error: RawOutput shouldn't be null");
+            return null;
+        }
+    }
+}

http://git-wip-us.apache.org/repos/asf/airavata/blob/cc1ca3d5/modules/worker/task-jobsubmission/src/main/java/org/apache/airavata/worker/task/jobsubmission/parser/UGEOutputParser.java
----------------------------------------------------------------------
diff --git a/modules/worker/task-jobsubmission/src/main/java/org/apache/airavata/worker/task/jobsubmission/parser/UGEOutputParser.java b/modules/worker/task-jobsubmission/src/main/java/org/apache/airavata/worker/task/jobsubmission/parser/UGEOutputParser.java
new file mode 100644
index 0000000..c3e7ba1
--- /dev/null
+++ b/modules/worker/task-jobsubmission/src/main/java/org/apache/airavata/worker/task/jobsubmission/parser/UGEOutputParser.java
@@ -0,0 +1,111 @@
+/*
+ *
+ * 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.airavata.worker.task.jobsubmission.parser;
+
+import org.apache.airavata.model.status.JobState;
+import org.apache.airavata.model.status.JobStatus;
+import org.apache.airavata.worker.core.cluster.OutputParser;
+import org.apache.airavata.worker.core.exceptions.WorkerException;
+import org.slf4j.Logger;
+import org.slf4j.LoggerFactory;
+
+import java.util.ArrayList;
+import java.util.List;
+import java.util.Map;
+import java.util.regex.Matcher;
+import java.util.regex.Pattern;
+
+public class UGEOutputParser implements OutputParser {
+    private static final Logger log = LoggerFactory.getLogger(PBSOutputParser.class);
+    public static final String JOB_ID = "jobId";
+
+	public String parseJobSubmission(String rawOutput) {
+		log.debug(rawOutput);
+		if (rawOutput != null && !rawOutput.isEmpty() && !isJobSubmissionFailed(rawOutput)) {
+			String[] info = rawOutput.split("\n");
+			String lastLine = info[info.length - 1];
+			return lastLine.split(" ")[2]; // In PBS stdout is going to be directly the jobID
+		} else {
+			return "";
+		}
+	}
+
+    @Override
+    public boolean isJobSubmissionFailed(String rawOutput) {
+        Pattern pattern = Pattern.compile("Rejecting");
+        Matcher matcher = pattern.matcher(rawOutput);
+        return matcher.find();
+    }
+
+    public JobStatus parseJobStatus(String jobID, String rawOutput) {
+        Pattern pattern = Pattern.compile("job_number:[\\s]+" + jobID);
+        Matcher matcher = pattern.matcher(rawOutput);
+        if (matcher.find()) {
+	        return new JobStatus(JobState.QUEUED); // fixme; return correct status.
+        }
+	    return new JobStatus(JobState.UNKNOWN);
+    }
+
+    public void parseJobStatuses(String userName, Map<String, JobStatus> statusMap, String rawOutput) {
+        log.debug(rawOutput);
+        String[] info = rawOutput.split("\n");
+        int lastStop = 0;
+        for (String jobID : statusMap.keySet()) {
+            for(int i=lastStop;i<info.length;i++){
+               if(jobID.split(",")[0].contains(info[i].split(" ")[0]) && !"".equals(info[i].split(" ")[0])){
+                   // now starts processing this line
+                   log.info(info[i]);
+                   String correctLine = info[i];
+                   String[] columns = correctLine.split(" ");
+                   List<String> columnList = new ArrayList<String>();
+                   for (String s : columns) {
+                       if (!"".equals(s)) {
+                           columnList.add(s);
+                       }
+                   }
+                   lastStop = i+1;
+                   if ("E".equals(columnList.get(4))) {
+                       // There is another status with the same letter E other than error status
+                       // to avoid that we make a small tweek to the job status
+                       columnList.set(4, "Er");
+                   }
+	               statusMap.put(jobID, new JobStatus(JobState.valueOf(columnList.get(4))));
+	               break;
+               }
+            }
+        }
+    }
+
+    @Override
+    public String parseJobId(String jobName, String rawOutput) throws WorkerException {
+        if (jobName.length() > 10) {
+            jobName = jobName.substring(0, 10);
+        }
+        Pattern pattern = Pattern.compile("(?<" + JOB_ID + ">\\S+)\\s+\\S+\\s+(" + jobName + ")");
+        Matcher matcher = pattern.matcher(rawOutput);
+        if (matcher.find()) {
+            return matcher.group(JOB_ID);
+        }
+        return null;
+    }
+
+
+}

http://git-wip-us.apache.org/repos/asf/airavata/blob/cc1ca3d5/modules/worker/task-jobsubmission/src/main/java/org/apache/airavata/worker/task/jobsubmission/utils/JobSubmissionUtils.java
----------------------------------------------------------------------
diff --git a/modules/worker/task-jobsubmission/src/main/java/org/apache/airavata/worker/task/jobsubmission/utils/JobSubmissionUtils.java b/modules/worker/task-jobsubmission/src/main/java/org/apache/airavata/worker/task/jobsubmission/utils/JobSubmissionUtils.java
index b4ebe1b..b6f867c 100644
--- a/modules/worker/task-jobsubmission/src/main/java/org/apache/airavata/worker/task/jobsubmission/utils/JobSubmissionUtils.java
+++ b/modules/worker/task-jobsubmission/src/main/java/org/apache/airavata/worker/task/jobsubmission/utils/JobSubmissionUtils.java
@@ -17,6 +17,7 @@ import org.apache.airavata.model.job.JobModel;
 import org.apache.airavata.model.parallelism.ApplicationParallelismType;
 import org.apache.airavata.model.process.ProcessModel;
 import org.apache.airavata.model.scheduling.ComputationalResourceSchedulingModel;
+import org.apache.airavata.model.status.JobState;
 import org.apache.airavata.model.task.JobSubmissionTaskModel;
 import org.apache.airavata.registry.core.experiment.catalog.impl.RegistryFactory;
 import org.apache.airavata.registry.cpi.*;
@@ -561,4 +562,30 @@ public class JobSubmissionUtils {
         outputPath = (outputPath.endsWith(File.separator) ? outputPath : outputPath + File.separator);
         return new File(outputPath + taskContext.getParentProcessContext() .getProcessId());
     }
+
+    public static JobState getJobState(String status) {
+        log.info("Parsing the job status returned: " + status);
+        if (status != null) {
+            if ("C".equals(status) || "CD".equals(status) || "E".equals(status) || "CG".equals(status) || "DONE".equals(status)) {
+                return JobState.COMPLETE;
+            } else if ("Q".equals(status) || "qw".equals(status) || "PEND".equals(status)) {
+                return JobState.QUEUED;
+            } else if ("R".equals(status) || "CF".equals(status) || "r".equals(status) || "RUN".equals(status)) {
+                return JobState.ACTIVE;
+            } else if ("W".equals(status) || "PD".equals(status)) {
+                return JobState.QUEUED;
+            } else if ("S".equals(status) || "PSUSP".equals(status) || "USUSP".equals(status) || "SSUSP".equals(status)) {
+                return JobState.SUSPENDED;
+            } else if ("CA".equals(status)) {
+                return JobState.CANCELED;
+            } else if ("F".equals(status) || "NF".equals(status) || "TO".equals(status) || "EXIT".equals(status)) {
+                return JobState.FAILED;
+            } else if ("PR".equals(status) || "Er".equals(status)) {
+                return JobState.FAILED;
+            } else if ("U".equals(status) || ("UNKWN".equals(status))) {
+                return JobState.UNKNOWN;
+            }
+        }
+        return JobState.UNKNOWN;
+    }
 }