|
||||||||||
PREV CLASS NEXT CLASS | FRAMES NO FRAMES | |||||||||
SUMMARY: NESTED | FIELD | CONSTR | METHOD | DETAIL: FIELD | CONSTR | METHOD |
java.lang.Object org.apache.hadoop.mapreduce.RecordReader<K,V> org.apache.hadoop.mapreduce.lib.join.ComposableRecordReader<K,X> org.apache.hadoop.mapreduce.lib.join.CompositeRecordReader<K,V,X>
@InterfaceAudience.Public @InterfaceStability.Stable public abstract class CompositeRecordReader<K extends WritableComparable<?>,V extends Writable,X extends Writable>
A RecordReader that can effect joins of RecordReaders sharing a common key type and partitioning.
Field Summary | |
---|---|
protected Configuration |
conf
|
protected org.apache.hadoop.mapreduce.lib.join.CompositeRecordReader.JoinCollector |
jc
|
protected K |
key
|
protected Class<? extends WritableComparable> |
keyclass
|
protected ComposableRecordReader<K,? extends V>[] |
kids
|
protected X |
value
|
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.mapreduce.lib.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 K |
createKey()
Create a new key common to all child RRs. |
protected TupleWritable |
createTupleWritable()
Create a value to be used internally for joins. |
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. |
K |
getCurrentKey()
Get the current key |
X |
getCurrentValue()
Get the current value. |
protected abstract ResetableIterator<X> |
getDelegate()
Obtain an iterator over the child RRs apropos of the value type ultimately emitted from this join. |
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. |
void |
initialize(InputSplit split,
TaskAttemptContext context)
Called once at initialization. |
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 org.apache.hadoop.mapreduce.RecordReader |
---|
nextKeyValue |
Methods inherited from class java.lang.Object |
---|
clone, equals, finalize, getClass, hashCode, notify, notifyAll, toString, wait, wait, wait |
Field Detail |
---|
protected Configuration conf
protected Class<? extends WritableComparable> keyclass
protected final org.apache.hadoop.mapreduce.lib.join.CompositeRecordReader.JoinCollector jc
protected final ComposableRecordReader<K extends WritableComparable<?>,? extends V extends Writable>[] kids
protected K extends WritableComparable<?> key
protected X extends Writable value
Constructor Detail |
---|
public CompositeRecordReader(int id, int capacity, Class<? extends WritableComparator> cmpcl) throws IOException
IOException
Method Detail |
---|
protected abstract boolean combine(Object[] srcs, TupleWritable value)
public void initialize(InputSplit split, TaskAttemptContext context) throws IOException, InterruptedException
RecordReader
initialize
in class RecordReader<K extends WritableComparable<?>,X extends Writable>
split
- the split that defines the range of records to readcontext
- the information about the task
IOException
InterruptedException
public int id()
public void setConf(Configuration conf)
setConf
in interface Configurable
public Configuration getConf()
getConf
in interface Configurable
protected PriorityQueue<ComposableRecordReader<K,?>> getRecordReaderQueue()
protected WritableComparator getComparator()
public void add(ComposableRecordReader<K,? extends V> rr) throws IOException, InterruptedException
IOException
InterruptedException
public K key()
public void key(K key) throws IOException
IOException
public K getCurrentKey()
RecordReader
getCurrentKey
in class RecordReader<K extends WritableComparable<?>,X extends Writable>
public boolean hasNext()
public void skip(K key) throws IOException, InterruptedException
IOException
InterruptedException
protected abstract ResetableIterator<X> getDelegate()
public void accept(org.apache.hadoop.mapreduce.lib.join.CompositeRecordReader.JoinCollector jc, K key) throws IOException, InterruptedException
IOException
InterruptedException
protected void fillJoinCollector(K iterkey) throws IOException, InterruptedException
IOException
InterruptedException
public int compareTo(ComposableRecordReader<K,?> other)
compareTo
in interface Comparable<ComposableRecordReader<K extends WritableComparable<?>,?>>
protected K createKey()
ClassCastException
- if key classes differ.protected TupleWritable createTupleWritable()
public X getCurrentValue() throws IOException, InterruptedException
getCurrentValue
in class RecordReader<K extends WritableComparable<?>,X extends Writable>
IOException
InterruptedException
public void close() throws IOException
close
in interface Closeable
close
in class RecordReader<K extends WritableComparable<?>,X extends Writable>
IOException
public float getProgress() throws IOException, InterruptedException
getProgress
in class RecordReader<K extends WritableComparable<?>,X extends Writable>
IOException
InterruptedException
|
||||||||||
PREV CLASS NEXT CLASS | FRAMES NO FRAMES | |||||||||
SUMMARY: NESTED | FIELD | CONSTR | METHOD | DETAIL: FIELD | CONSTR | METHOD |