org.apache.hadoop.mapred.join
Class CompositeRecordReader<K extends WritableComparable,V extends Writable,X extends Writable>

java.lang.Object
  extended by org.apache.hadoop.mapred.join.CompositeRecordReader<K,V,X>
All Implemented Interfaces:
Configurable
Direct Known Subclasses:
JoinRecordReader, MultiFilterRecordReader

@InterfaceAudience.Public
@InterfaceStability.Stable
public abstract class CompositeRecordReader<K extends WritableComparable,V extends Writable,X extends Writable>
extends Object
implements Configurable

A RecordReader that can effect joins of RecordReaders sharing a common key type and partitioning.


Field Summary
protected  org.apache.hadoop.mapred.join.CompositeRecordReader.JoinCollector jc
           
protected  ComposableRecordReader<K,? extends V>[] kids
           
 
Constructor Summary
CompositeRecordReader(int id, int capacity, Class<? extends WritableComparator> cmpcl)
          Create a RecordReader with capacity children to position id in the parent reader.
 
Method Summary
 void accept(org.apache.hadoop.mapred.join.CompositeRecordReader.JoinCollector jc, K key)
          If key provided matches that of this Composite, give JoinCollector iterator over values it may emit.
 void add(ComposableRecordReader<K,? extends V> rr)
          Add a RecordReader to this collection.
 void close()
          Close all child RRs.
protected abstract  boolean combine(Object[] srcs, TupleWritable value)
           
 int compareTo(ComposableRecordReader<K,?> other)
          Implement Comparable contract (compare key of join or head of heap with that of another).
protected  TupleWritable createInternalValue()
          Create a value to be used internally for joins.
 K createKey()
          Create a new key value common to all child RRs.
protected  void fillJoinCollector(K iterkey)
          For all child RRs offering the key provided, obtain an iterator at that position in the JoinCollector.
protected  WritableComparator getComparator()
          Return comparator defining the ordering for RecordReaders in this composite.
 Configuration getConf()
          Return the configuration used by this object.
protected abstract  ResetableIterator<X> getDelegate()
          Obtain an iterator over the child RRs apropos of the value type ultimately emitted from this join.
 long getPos()
          Unsupported (returns zero in all cases).
 float getProgress()
          Report progress as the minimum of all child RR progress.
protected  PriorityQueue<ComposableRecordReader<K,?>> getRecordReaderQueue()
          Return sorted list of RecordReaders for this composite.
 boolean hasNext()
          Return true if it is possible that this could emit more values.
 int id()
          Return the position in the collector this class occupies.
 K key()
          Return the key for the current join or the value at the top of the RecordReader heap.
 void key(K key)
          Clone the key at the top of this RR into the given object.
 void setConf(Configuration conf)
          Set the configuration to be used by this object.
 void skip(K key)
          Pass skip key to child RRs.
 
Methods inherited from class java.lang.Object
clone, equals, finalize, getClass, hashCode, notify, notifyAll, toString, wait, wait, wait
 

Field Detail

jc

protected final org.apache.hadoop.mapred.join.CompositeRecordReader.JoinCollector jc

kids

protected final ComposableRecordReader<K extends WritableComparable,? extends V extends Writable>[] kids
Constructor Detail

CompositeRecordReader

public CompositeRecordReader(int id,
                             int capacity,
                             Class<? extends WritableComparator> cmpcl)
                      throws IOException
Create a RecordReader with capacity children to position id in the parent reader. The id of a root CompositeRecordReader is -1 by convention, but relying on this is not recommended.

Throws:
IOException
Method Detail

combine

protected abstract boolean combine(Object[] srcs,
                                   TupleWritable value)

id

public int id()
Return the position in the collector this class occupies.


setConf

public void setConf(Configuration conf)
Set the configuration to be used by this object.

Specified by:
setConf in interface Configurable

getConf

public Configuration getConf()
Return the configuration used by this object.

Specified by:
getConf in interface Configurable

getRecordReaderQueue

protected PriorityQueue<ComposableRecordReader<K,?>> getRecordReaderQueue()
Return sorted list of RecordReaders for this composite.


getComparator

protected WritableComparator getComparator()
Return comparator defining the ordering for RecordReaders in this composite.


add

public void add(ComposableRecordReader<K,? extends V> rr)
         throws IOException
Add a RecordReader to this collection. The id() of a RecordReader determines where in the Tuple its entry will appear. Adding RecordReaders with the same id has undefined behavior.

Throws:
IOException

key

public K key()
Return the key for the current join or the value at the top of the RecordReader heap.


key

public void key(K key)
         throws IOException
Clone the key at the top of this RR into the given object.

Throws:
IOException

hasNext

public boolean hasNext()
Return true if it is possible that this could emit more values.


skip

public void skip(K key)
          throws IOException
Pass skip key to child RRs.

Throws:
IOException

getDelegate

protected abstract ResetableIterator<X> getDelegate()
Obtain an iterator over the child RRs apropos of the value type ultimately emitted from this join.


accept

public void accept(org.apache.hadoop.mapred.join.CompositeRecordReader.JoinCollector jc,
                   K key)
            throws IOException
If key provided matches that of this Composite, give JoinCollector iterator over values it may emit.

Throws:
IOException

fillJoinCollector

protected void fillJoinCollector(K iterkey)
                          throws IOException
For all child RRs offering the key provided, obtain an iterator at that position in the JoinCollector.

Throws:
IOException

compareTo

public int compareTo(ComposableRecordReader<K,?> other)
Implement Comparable contract (compare key of join or head of heap with that of another).


createKey

public K createKey()
Create a new key value common to all child RRs.

Throws:
ClassCastException - if key classes differ.

createInternalValue

protected TupleWritable createInternalValue()
Create a value to be used internally for joins.


getPos

public long getPos()
            throws IOException
Unsupported (returns zero in all cases).

Throws:
IOException

close

public void close()
           throws IOException
Close all child RRs.

Throws:
IOException

getProgress

public float getProgress()
                  throws IOException
Report progress as the minimum of all child RR progress.

Throws:
IOException


Copyright © 2014 Apache Software Foundation. All Rights Reserved.