Updated Branches: refs/heads/master 99fbc8ee8 -> 0cbccccac
CRUNCH-332: Ensure DoFn.configure is called prior to DoFn.setContext and DoFn.initialize in the in-memory impl Project: http://git-wip-us.apache.org/repos/asf/crunch/repo Commit: http://git-wip-us.apache.org/repos/asf/crunch/commit/0cbcccca Tree: http://git-wip-us.apache.org/repos/asf/crunch/tree/0cbcccca Diff: http://git-wip-us.apache.org/repos/asf/crunch/diff/0cbcccca Branch: refs/heads/master Commit: 0cbccccac919989b9db9d870cef6a3e3c5d1cf4b Parents: 99fbc8e Author: Josh Wills <[email protected]> Authored: Fri Jan 24 08:31:10 2014 -0800 Committer: Josh Wills <[email protected]> Committed: Fri Jan 24 08:31:10 2014 -0800 ---------------------------------------------------------------------- .../org/apache/crunch/impl/mem/collect/MemCollection.java | 8 ++++++-- 1 file changed, 6 insertions(+), 2 deletions(-) ---------------------------------------------------------------------- http://git-wip-us.apache.org/repos/asf/crunch/blob/0cbcccca/crunch-core/src/main/java/org/apache/crunch/impl/mem/collect/MemCollection.java ---------------------------------------------------------------------- diff --git a/crunch-core/src/main/java/org/apache/crunch/impl/mem/collect/MemCollection.java b/crunch-core/src/main/java/org/apache/crunch/impl/mem/collect/MemCollection.java index bbcdc0b..81433eb 100644 --- a/crunch-core/src/main/java/org/apache/crunch/impl/mem/collect/MemCollection.java +++ b/crunch-core/src/main/java/org/apache/crunch/impl/mem/collect/MemCollection.java @@ -112,7 +112,9 @@ public class MemCollection<S> implements PCollection<S> { public <T> PCollection<T> parallelDo(String name, DoFn<S, T> doFn, PType<T> type, ParallelDoOptions options) { InMemoryEmitter<T> emitter = new InMemoryEmitter<T>(); - doFn.setContext(getInMemoryContext(getPipeline().getConfiguration())); + Configuration conf = getPipeline().getConfiguration(); + doFn.configure(conf); + doFn.setContext(getInMemoryContext(conf)); doFn.initialize(); for (S s : collect) { doFn.process(s, emitter); @@ -135,7 +137,9 @@ public class MemCollection<S> implements PCollection<S> { public <K, V> PTable<K, V> parallelDo(String name, DoFn<S, Pair<K, V>> doFn, PTableType<K, V> type, ParallelDoOptions options) { InMemoryEmitter<Pair<K, V>> emitter = new InMemoryEmitter<Pair<K, V>>(); - doFn.setContext(getInMemoryContext(getPipeline().getConfiguration())); + Configuration conf = getPipeline().getConfiguration(); + doFn.configure(conf); + doFn.setContext(getInMemoryContext(conf)); doFn.initialize(); for (S s : collect) { doFn.process(s, emitter);
