org.apache.hadoop.mapreduce
Class TaskInputOutputContext<KEYIN,VALUEIN,KEYOUT,VALUEOUT>

java.lang.Object
  extended by org.apache.hadoop.mapreduce.JobContext
      extended by org.apache.hadoop.mapreduce.TaskAttemptContext
          extended by org.apache.hadoop.mapreduce.TaskInputOutputContext<KEYIN,VALUEIN,KEYOUT,VALUEOUT>
Type Parameters:
KEYIN - the input key type for the task
VALUEIN - the input value type for the task
KEYOUT - the output key type for the task
VALUEOUT - the output value type for the task
All Implemented Interfaces:
Progressable
Direct Known Subclasses:
MapContext, ReduceContext

public abstract class TaskInputOutputContext<KEYIN,VALUEIN,KEYOUT,VALUEOUT>
extends TaskAttemptContext
implements Progressable

A context object that allows input and output from the task. It is only supplied to the Mapper or Reducer.


Field Summary
 
Fields inherited from class org.apache.hadoop.mapreduce.JobContext
CACHE_ARCHIVES_VISIBILITIES, CACHE_FILE_VISIBILITIES, COMBINE_CLASS_ATTR, conf, credentials, INPUT_FORMAT_CLASS_ATTR, JOB_ACL_MODIFY_JOB, JOB_ACL_VIEW_JOB, JOB_CANCEL_DELEGATION_TOKEN, JOB_NAMENODES, MAP_CLASS_ATTR, OUTPUT_FORMAT_CLASS_ATTR, PARTITIONER_CLASS_ATTR, REDUCE_CLASS_ATTR, ugi, USER_LOG_RETAIN_HOURS
 
Constructor Summary
TaskInputOutputContext(Configuration conf, TaskAttemptID taskid, RecordWriter<KEYOUT,VALUEOUT> output, OutputCommitter committer, StatusReporter reporter)
           
 
Method Summary
 Counter getCounter(Enum<?> counterName)
           
 Counter getCounter(String groupName, String counterName)
           
abstract  KEYIN getCurrentKey()
          Get the current key.
abstract  VALUEIN getCurrentValue()
          Get the current value.
 OutputCommitter getOutputCommitter()
           
 float getProgress()
           
abstract  boolean nextKeyValue()
          Advance to the next key, value pair, returning null if at end.
 void progress()
          Report progress.
 void setStatus(String status)
          Set the current status of the task to the given string.
 void write(KEYOUT key, VALUEOUT value)
          Generate an output key/value pair.
 
Methods inherited from class org.apache.hadoop.mapreduce.TaskAttemptContext
getStatus, getTaskAttemptID
 
Methods inherited from class org.apache.hadoop.mapreduce.JobContext
getCombinerClass, getConfiguration, getCredentials, getGroupingComparator, getInputFormatClass, getJar, getJobID, getJobName, getMapOutputKeyClass, getMapOutputValueClass, getMapperClass, getNumReduceTasks, getOutputFormatClass, getOutputKeyClass, getOutputValueClass, getPartitionerClass, getReducerClass, getSortComparator, getWorkingDirectory
 
Methods inherited from class java.lang.Object
clone, equals, finalize, getClass, hashCode, notify, notifyAll, toString, wait, wait, wait
 

Constructor Detail

TaskInputOutputContext

public TaskInputOutputContext(Configuration conf,
                              TaskAttemptID taskid,
                              RecordWriter<KEYOUT,VALUEOUT> output,
                              OutputCommitter committer,
                              StatusReporter reporter)
Method Detail

nextKeyValue

public abstract boolean nextKeyValue()
                              throws IOException,
                                     InterruptedException
Advance to the next key, value pair, returning null if at end.

Returns:
the key object that was read into, or null if no more
Throws:
IOException
InterruptedException

getCurrentKey

public abstract KEYIN getCurrentKey()
                             throws IOException,
                                    InterruptedException
Get the current key.

Returns:
the current key object or null if there isn't one
Throws:
IOException
InterruptedException

getCurrentValue

public abstract VALUEIN getCurrentValue()
                                 throws IOException,
                                        InterruptedException
Get the current value.

Returns:
the value object that was read into
Throws:
IOException
InterruptedException

write

public void write(KEYOUT key,
                  VALUEOUT value)
           throws IOException,
                  InterruptedException
Generate an output key/value pair.

Throws:
IOException
InterruptedException

getCounter

public Counter getCounter(Enum<?> counterName)

getCounter

public Counter getCounter(String groupName,
                          String counterName)

progress

public void progress()
Description copied from class: TaskAttemptContext
Report progress. The subtypes actually do work in this method.

Specified by:
progress in interface Progressable
Overrides:
progress in class TaskAttemptContext

getProgress

public float getProgress()

setStatus

public void setStatus(String status)
Description copied from class: TaskAttemptContext
Set the current status of the task to the given string.

Overrides:
setStatus in class TaskAttemptContext

getOutputCommitter

public OutputCommitter getOutputCommitter()


Copyright © 2009 The Apache Software Foundation