Support flat mapping with multiple threads (#1098)

Signed-off-by: Adrian Sutton <adrian.sutton@consensys.net>
pull/2/head
Adrian Sutton 6 years ago committed by GitHub
parent 6e8fa9ecc9
commit acd977668b
  1. 60
      services/pipeline/src/main/java/tech/pegasys/pantheon/services/pipeline/PipelineBuilder.java
  2. 38
      services/pipeline/src/test/java/tech/pegasys/pantheon/services/pipeline/PipelineBuilderTest.java

@ -26,6 +26,7 @@ import java.util.List;
import java.util.concurrent.CompletableFuture; import java.util.concurrent.CompletableFuture;
import java.util.function.Consumer; import java.util.function.Consumer;
import java.util.function.Function; import java.util.function.Function;
import java.util.function.Supplier;
import java.util.stream.Stream; import java.util.stream.Stream;
/** /**
@ -119,8 +120,8 @@ public class PipelineBuilder<T> {
} }
/** /**
* Adds a 1-to-1 processing stage to the pipeline. Multiple threads processes items in the * Adds a 1-to-1 processing stage to the pipeline. Multiple threads process items in the pipeline
* pipeline concurrently with <i>processor</i> outputting its return value to the next stage. * concurrently with <i>processor</i> outputting its return value to the next stage.
* *
* <p>Note: The order of items is not preserved. * <p>Note: The order of items is not preserved.
* *
@ -132,16 +133,8 @@ public class PipelineBuilder<T> {
*/ */
public <O> PipelineBuilder<O> thenProcessInParallel( public <O> PipelineBuilder<O> thenProcessInParallel(
final String stageName, final Function<T, O> processor, final int numberOfThreads) { final String stageName, final Function<T, O> processor, final int numberOfThreads) {
final Pipe<O> newPipeEnd = new Pipe<>(bufferSize, outputCounter.labels(stageName)); return thenProcessInParallel(
final WritePipe<O> outputPipe = new SharedWritePipe<>(newPipeEnd, numberOfThreads); stageName, () -> new MapProcessor<>(processor), numberOfThreads, bufferSize);
final ArrayList<Stage> newStages = new ArrayList<>(stages);
for (int i = 0; i < numberOfThreads; i++) {
final Stage processStage =
new ProcessingStage<>(stageName, pipeEnd, outputPipe, new MapProcessor<>(processor));
newStages.add(processStage);
}
return new PipelineBuilder<>(
inputPipe, newStages, concat(pipes, newPipeEnd), newPipeEnd, bufferSize, outputCounter);
} }
/** /**
@ -206,6 +199,32 @@ public class PipelineBuilder<T> {
return addStage(new FlatMapProcessor<>(mapper), newBufferSize, stageName); return addStage(new FlatMapProcessor<>(mapper), newBufferSize, stageName);
} }
/**
* Adds a 1-to-many processing stage to the pipeline. For each item in the stream, <i>mapper</i>
* is called and each item of the {@link Stream} it returns is output as an individual item. The
* returned Stream may be empty to remove an item. Multiple threads process items in the pipeline
* concurrently.
*
* <p>This can be used to reverse the effect of {@link #inBatches(String, int)} with:
*
* <pre>thenFlatMap(List::stream, newBufferSize)</pre>
*
* @param stageName the name of this stage. Used as the label for the output count metric.
* @param mapper the function to process each item with.
* @param numberOfThreads the number of threads to use for processing.
* @param newBufferSize the output buffer size to use from this stage onwards.
* @param <O> the type of items to be output from this stage.
* @return a {@link PipelineBuilder} ready to extend the pipeline with additional stages.
*/
public <O> PipelineBuilder<O> thenFlatMapInParallel(
final String stageName,
final Function<T, Stream<O>> mapper,
final int numberOfThreads,
final int newBufferSize) {
return thenProcessInParallel(
stageName, () -> new FlatMapProcessor<>(mapper), numberOfThreads, newBufferSize);
}
/** /**
* End the pipeline with a {@link Consumer} that is the last stage of the pipeline. * End the pipeline with a {@link Consumer} that is the last stage of the pipeline.
* *
@ -221,6 +240,23 @@ public class PipelineBuilder<T> {
new CompleterStage<>(stageName, pipeEnd, completer, outputCounter.labels(stageName))); new CompleterStage<>(stageName, pipeEnd, completer, outputCounter.labels(stageName)));
} }
private <O> PipelineBuilder<O> thenProcessInParallel(
final String stageName,
final Supplier<Processor<T, O>> createProcessor,
final int numberOfThreads,
final int newBufferSize) {
final Pipe<O> newPipeEnd = new Pipe<>(newBufferSize, outputCounter.labels(stageName));
final WritePipe<O> outputPipe = new SharedWritePipe<>(newPipeEnd, numberOfThreads);
final ArrayList<Stage> newStages = new ArrayList<>(stages);
for (int i = 0; i < numberOfThreads; i++) {
final Stage processStage =
new ProcessingStage<>(stageName, pipeEnd, outputPipe, createProcessor.get());
newStages.add(processStage);
}
return new PipelineBuilder<>(
inputPipe, newStages, concat(pipes, newPipeEnd), newPipeEnd, newBufferSize, outputCounter);
}
private <O> PipelineBuilder<O> addStage(final Processor<T, O> processor, final String stageName) { private <O> PipelineBuilder<O> addStage(final Processor<T, O> processor, final String stageName) {
return addStage(processor, bufferSize, stageName); return addStage(processor, bufferSize, stageName);
} }

@ -232,6 +232,44 @@ public class PipelineBuilderTest {
"1", "3", "4", "5", "6", "7", "8", "9", "10", "11", "12", "13", "14", "15", "2"); "1", "3", "4", "5", "6", "7", "8", "9", "10", "11", "12", "13", "14", "15", "2");
} }
@Test
public void shouldFlatMapInParallel() throws Exception {
final List<String> output = synchronizedList(new ArrayList<>());
final CountDownLatch latch = new CountDownLatch(1);
final Pipeline pipeline =
PipelineBuilder.createPipelineFrom("input", tasks, 10, NO_OP_LABELLED_COUNTER)
.thenFlatMapInParallel(
"stageName",
value -> {
if (value == 2) {
try {
latch.await();
} catch (InterruptedException e) {
throw new RuntimeException(e);
}
}
return Stream.of(value.toString(), "x" + value);
},
2,
10)
.andFinishWith("end", output::add);
final CompletableFuture<?> result = pipeline.start(executorService);
// One thread will block but the other should process the remaining entries.
waitForSize(output, 28);
assertThat(result).isNotDone();
latch.countDown();
result.get(10, SECONDS);
assertThat(output)
.containsExactly(
"1", "x1", "3", "x3", "4", "x4", "5", "x5", "6", "x6", "7", "x7", "8", "x8", "9", "x9",
"10", "x10", "11", "x11", "12", "x12", "13", "x13", "14", "x14", "15", "x15", "2",
"x2");
}
@Test @Test
public void shouldAbortPipeline() throws Exception { public void shouldAbortPipeline() throws Exception {
final int allowProcessingUpTo = 5; final int allowProcessingUpTo = 5;

Loading…
Cancel
Save