/
Combine.java
2239 lines (1984 loc) · 86.6 KB
/
Combine.java
1
2
3
4
5
6
7
8
9
10
11
12
13
14
15
16
17
18
19
20
21
22
23
24
25
26
27
28
29
30
31
32
33
34
35
36
37
38
39
40
41
42
43
44
45
46
47
48
49
50
51
52
53
54
55
56
57
58
59
60
61
62
63
64
65
66
67
68
69
70
71
72
73
74
75
76
77
78
79
80
81
82
83
84
85
86
87
88
89
90
91
92
93
94
95
96
97
98
99
100
101
102
103
104
105
106
107
108
109
110
111
112
113
114
115
116
117
118
119
120
121
122
123
124
125
126
127
128
129
130
131
132
133
134
135
136
137
138
139
140
141
142
143
144
145
146
147
148
149
150
151
152
153
154
155
156
157
158
159
160
161
162
163
164
165
166
167
168
169
170
171
172
173
174
175
176
177
178
179
180
181
182
183
184
185
186
187
188
189
190
191
192
193
194
195
196
197
198
199
200
201
202
203
204
205
206
207
208
209
210
211
212
213
214
215
216
217
218
219
220
221
222
223
224
225
226
227
228
229
230
231
232
233
234
235
236
237
238
239
240
241
242
243
244
245
246
247
248
249
250
251
252
253
254
255
256
257
258
259
260
261
262
263
264
265
266
267
268
269
270
271
272
273
274
275
276
277
278
279
280
281
282
283
284
285
286
287
288
289
290
291
292
293
294
295
296
297
298
299
300
301
302
303
304
305
306
307
308
309
310
311
312
313
314
315
316
317
318
319
320
321
322
323
324
325
326
327
328
329
330
331
332
333
334
335
336
337
338
339
340
341
342
343
344
345
346
347
348
349
350
351
352
353
354
355
356
357
358
359
360
361
362
363
364
365
366
367
368
369
370
371
372
373
374
375
376
377
378
379
380
381
382
383
384
385
386
387
388
389
390
391
392
393
394
395
396
397
398
399
400
401
402
403
404
405
406
407
408
409
410
411
412
413
414
415
416
417
418
419
420
421
422
423
424
425
426
427
428
429
430
431
432
433
434
435
436
437
438
439
440
441
442
443
444
445
446
447
448
449
450
451
452
453
454
455
456
457
458
459
460
461
462
463
464
465
466
467
468
469
470
471
472
473
474
475
476
477
478
479
480
481
482
483
484
485
486
487
488
489
490
491
492
493
494
495
496
497
498
499
500
501
502
503
504
505
506
507
508
509
510
511
512
513
514
515
516
517
518
519
520
521
522
523
524
525
526
527
528
529
530
531
532
533
534
535
536
537
538
539
540
541
542
543
544
545
546
547
548
549
550
551
552
553
554
555
556
557
558
559
560
561
562
563
564
565
566
567
568
569
570
571
572
573
574
575
576
577
578
579
580
581
582
583
584
585
586
587
588
589
590
591
592
593
594
595
596
597
598
599
600
601
602
603
604
605
606
607
608
609
610
611
612
613
614
615
616
617
618
619
620
621
622
623
624
625
626
627
628
629
630
631
632
633
634
635
636
637
638
639
640
641
642
643
644
645
646
647
648
649
650
651
652
653
654
655
656
657
658
659
660
661
662
663
664
665
666
667
668
669
670
671
672
673
674
675
676
677
678
679
680
681
682
683
684
685
686
687
688
689
690
691
692
693
694
695
696
697
698
699
700
701
702
703
704
705
706
707
708
709
710
711
712
713
714
715
716
717
718
719
720
721
722
723
724
725
726
727
728
729
730
731
732
733
734
735
736
737
738
739
740
741
742
743
744
745
746
747
748
749
750
751
752
753
754
755
756
757
758
759
760
761
762
763
764
765
766
767
768
769
770
771
772
773
774
775
776
777
778
779
780
781
782
783
784
785
786
787
788
789
790
791
792
793
794
795
796
797
798
799
800
801
802
803
804
805
806
807
808
809
810
811
812
813
814
815
816
817
818
819
820
821
822
823
824
825
826
827
828
829
830
831
832
833
834
835
836
837
838
839
840
841
842
843
844
845
846
847
848
849
850
851
852
853
854
855
856
857
858
859
860
861
862
863
864
865
866
867
868
869
870
871
872
873
874
875
876
877
878
879
880
881
882
883
884
885
886
887
888
889
890
891
892
893
894
895
896
897
898
899
900
901
902
903
904
905
906
907
908
909
910
911
912
913
914
915
916
917
918
919
920
921
922
923
924
925
926
927
928
929
930
931
932
933
934
935
936
937
938
939
940
941
942
943
944
945
946
947
948
949
950
951
952
953
954
955
956
957
958
959
960
961
962
963
964
965
966
967
968
969
970
971
972
973
974
975
976
977
978
979
980
981
982
983
984
985
986
987
988
989
990
991
992
993
994
995
996
997
998
999
1000
/*
* Licensed to the Apache Software Foundation (ASF) under one
* or more contributor license agreements. See the NOTICE file
* distributed with this work for additional information
* regarding copyright ownership. The ASF licenses this file
* to you under the Apache License, Version 2.0 (the
* "License"); you may not use this file except in compliance
* with the License. You may obtain a copy of the License at
*
* http://www.apache.org/licenses/LICENSE-2.0
*
* Unless required by applicable law or agreed to in writing, software
* distributed under the License is distributed on an "AS IS" BASIS,
* WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
* See the License for the specific language governing permissions and
* limitations under the License.
*/
package org.apache.beam.sdk.transforms;
import static org.apache.beam.vendor.guava.v26_0_jre.com.google.common.base.Preconditions.checkState;
import java.io.IOException;
import java.io.InputStream;
import java.io.OutputStream;
import java.util.ArrayList;
import java.util.Arrays;
import java.util.Collections;
import java.util.Iterator;
import java.util.List;
import java.util.Map;
import java.util.concurrent.ThreadLocalRandom;
import javax.annotation.Nullable;
import org.apache.beam.sdk.coders.CannotProvideCoderException;
import org.apache.beam.sdk.coders.Coder;
import org.apache.beam.sdk.coders.CoderException;
import org.apache.beam.sdk.coders.CoderRegistry;
import org.apache.beam.sdk.coders.DelegateCoder;
import org.apache.beam.sdk.coders.IterableCoder;
import org.apache.beam.sdk.coders.KvCoder;
import org.apache.beam.sdk.coders.StructuredCoder;
import org.apache.beam.sdk.coders.VarIntCoder;
import org.apache.beam.sdk.coders.VoidCoder;
import org.apache.beam.sdk.options.PipelineOptions;
import org.apache.beam.sdk.transforms.CombineFnBase.AbstractGlobalCombineFn;
import org.apache.beam.sdk.transforms.CombineFnBase.GlobalCombineFn;
import org.apache.beam.sdk.transforms.CombineWithContext.CombineFnWithContext;
import org.apache.beam.sdk.transforms.CombineWithContext.Context;
import org.apache.beam.sdk.transforms.CombineWithContext.RequiresContextInternal;
import org.apache.beam.sdk.transforms.View.CreatePCollectionView;
import org.apache.beam.sdk.transforms.View.VoidKeyToMultimapMaterialization;
import org.apache.beam.sdk.transforms.display.DisplayData;
import org.apache.beam.sdk.transforms.display.DisplayData.Builder;
import org.apache.beam.sdk.transforms.display.HasDisplayData;
import org.apache.beam.sdk.transforms.windowing.GlobalWindow;
import org.apache.beam.sdk.transforms.windowing.GlobalWindows;
import org.apache.beam.sdk.transforms.windowing.Window;
import org.apache.beam.sdk.util.AppliedCombineFn;
import org.apache.beam.sdk.util.NameUtils;
import org.apache.beam.sdk.util.NameUtils.NameOverride;
import org.apache.beam.sdk.util.SerializableUtils;
import org.apache.beam.sdk.values.KV;
import org.apache.beam.sdk.values.PCollection;
import org.apache.beam.sdk.values.PCollectionList;
import org.apache.beam.sdk.values.PCollectionTuple;
import org.apache.beam.sdk.values.PCollectionView;
import org.apache.beam.sdk.values.PCollectionViews;
import org.apache.beam.sdk.values.PCollectionViews.TypeDescriptorSupplier;
import org.apache.beam.sdk.values.PValue;
import org.apache.beam.sdk.values.TupleTag;
import org.apache.beam.sdk.values.TupleTagList;
import org.apache.beam.sdk.values.TypeDescriptor;
import org.apache.beam.sdk.values.WindowingStrategy;
import org.apache.beam.vendor.guava.v26_0_jre.com.google.common.collect.ImmutableList;
import org.apache.beam.vendor.guava.v26_0_jre.com.google.common.collect.Iterables;
/**
* {@code PTransform}s for combining {@code PCollection} elements globally and per-key.
*
* <p>See the <a
* href="https://beam.apache.org/documentation/programming-guide/#transforms-combine">documentation</a>
* for how to use the operations in this class.
*/
public class Combine {
private Combine() {
// do not instantiate
}
/**
* Returns a {@link Globally Combine.Globally} {@code PTransform} that uses the given {@code
* SerializableFunction} to combine all the elements in each window of the input {@code
* PCollection} into a single value in the output {@code PCollection}. The types of the input
* elements and the output elements must be the same.
*
* <p>If the input {@code PCollection} is windowed into {@link GlobalWindows}, a default value in
* the {@link GlobalWindow} will be output if the input {@code PCollection} is empty. To use this
* with inputs with other windowing, either {@link Globally#withoutDefaults} or {@link
* Globally#asSingletonView} must be called.
*
* <p>See {@link Globally Combine.Globally} for more information.
*/
public static <V> Globally<V, V> globally(SerializableFunction<Iterable<V>, V> combiner) {
return globally(IterableCombineFn.of(combiner), displayDataForFn(combiner));
}
/**
* Returns a {@link Globally Combine.Globally} {@code PTransform} that uses the given {@code
* SerializableBiFunction} to combine all the elements in each window of the input {@code
* PCollection} into a single value in the output {@code PCollection}. The types of the input
* elements and the output elements must be the same.
*
* <p>If the input {@code PCollection} is windowed into {@link GlobalWindows}, a default value in
* the {@link GlobalWindow} will be output if the input {@code PCollection} is empty. To use this
* with inputs with other windowing, either {@link Globally#withoutDefaults} or {@link
* Globally#asSingletonView} must be called.
*
* <p>See {@link Globally Combine.Globally} for more information.
*/
public static <V> Globally<V, V> globally(SerializableBiFunction<V, V, V> combiner) {
return globally(BinaryCombineFn.of(combiner), displayDataForFn(combiner));
}
/**
* Returns a {@link Globally Combine.Globally} {@code PTransform} that uses the given {@code
* GloballyCombineFn} to combine all the elements in each window of the input {@code PCollection}
* into a single value in the output {@code PCollection}. The types of the input elements and the
* output elements can differ.
*
* <p>If the input {@code PCollection} is windowed into {@link GlobalWindows}, a default value in
* the {@link GlobalWindow} will be output if the input {@code PCollection} is empty. To use this
* with inputs with other windowing, either {@link Globally#withoutDefaults} or {@link
* Globally#asSingletonView} must be called.
*
* <p>See {@link Globally Combine.Globally} for more information.
*/
public static <InputT, OutputT> Globally<InputT, OutputT> globally(
GlobalCombineFn<? super InputT, ?, OutputT> fn) {
return globally(fn, displayDataForFn(fn));
}
private static <T> DisplayData.ItemSpec<? extends Class<?>> displayDataForFn(T fn) {
return DisplayData.item("combineFn", fn.getClass()).withLabel("Combiner");
}
private static <InputT, OutputT> Globally<InputT, OutputT> globally(
GlobalCombineFn<? super InputT, ?, OutputT> fn,
DisplayData.ItemSpec<? extends Class<?>> fnDisplayData) {
return new Globally<>(fn, fnDisplayData, true, 0, ImmutableList.of());
}
/**
* Returns a {@link PerKey Combine.PerKey} {@code PTransform} that first groups its input {@code
* PCollection} of {@code KV}s by keys and windows, then invokes the given function on each of the
* values lists to produce a combined value, and then returns a {@code PCollection} of {@code KV}s
* mapping each distinct key to its combined value for each window.
*
* <p>Each output element is in the window by which its corresponding input was grouped, and has
* the timestamp of the end of that window. The output {@code PCollection} has the same {@link
* org.apache.beam.sdk.transforms.windowing.WindowFn} as the input.
*
* <p>See {@link PerKey Combine.PerKey} for more information.
*/
public static <K, V> PerKey<K, V, V> perKey(SerializableFunction<Iterable<V>, V> fn) {
return perKey(IterableCombineFn.of(fn), displayDataForFn(fn));
}
/**
* Returns a {@link PerKey Combine.PerKey} {@code PTransform} that first groups its input {@code
* PCollection} of {@code KV}s by keys and windows, then invokes the given function on each of the
* values lists to produce a combined value, and then returns a {@code PCollection} of {@code KV}s
* mapping each distinct key to its combined value for each window.
*
* <p>Each output element is in the window by which its corresponding input was grouped, and has
* the timestamp of the end of that window. The output {@code PCollection} has the same {@link
* org.apache.beam.sdk.transforms.windowing.WindowFn} as the input.
*
* <p>See {@link PerKey Combine.PerKey} for more information.
*/
public static <K, V> PerKey<K, V, V> perKey(SerializableBiFunction<V, V, V> fn) {
return perKey(BinaryCombineFn.of(fn), displayDataForFn(fn));
}
/**
* Returns a {@link PerKey Combine.PerKey} {@code PTransform} that first groups its input {@code
* PCollection} of {@code KV}s by keys and windows, then invokes the given function on each of the
* values lists to produce a combined value, and then returns a {@code PCollection} of {@code KV}s
* mapping each distinct key to its combined value for each window.
*
* <p>Each output element is in the window by which its corresponding input was grouped, and has
* the timestamp of the end of that window. The output {@code PCollection} has the same {@link
* org.apache.beam.sdk.transforms.windowing.WindowFn} as the input.
*
* <p>See {@link PerKey Combine.PerKey} for more information.
*/
public static <K, InputT, OutputT> PerKey<K, InputT, OutputT> perKey(
GlobalCombineFn<? super InputT, ?, OutputT> fn) {
return perKey(fn, displayDataForFn(fn));
}
private static <K, InputT, OutputT> PerKey<K, InputT, OutputT> perKey(
GlobalCombineFn<? super InputT, ?, OutputT> fn,
DisplayData.ItemSpec<? extends Class<?>> fnDisplayData) {
return new PerKey<>(fn, fnDisplayData, false /*fewKeys*/);
}
/** Returns a {@link PerKey Combine.PerKey}, and set fewKeys in {@link GroupByKey}. */
private static <K, InputT, OutputT> PerKey<K, InputT, OutputT> fewKeys(
GlobalCombineFn<? super InputT, ?, OutputT> fn,
DisplayData.ItemSpec<? extends Class<?>> fnDisplayData) {
return new PerKey<>(fn, fnDisplayData, true /*fewKeys*/);
}
/**
* Returns a {@link GroupedValues Combine.GroupedValues} {@code PTransform} that takes a {@code
* PCollection} of {@code KV}s where a key maps to an {@code Iterable} of values, e.g., the result
* of a {@code GroupByKey}, then uses the given {@code SerializableFunction} to combine all the
* values associated with a key, ignoring the key. The type of the input and output values must be
* the same.
*
* <p>Each output element has the same timestamp and is in the same window as its corresponding
* input element, and the output {@code PCollection} has the same {@link
* org.apache.beam.sdk.transforms.windowing.WindowFn} associated with it as the input.
*
* <p>See {@link GroupedValues Combine.GroupedValues} for more information.
*
* <p>Note that {@link #perKey(SerializableFunction)} is typically more convenient to use than
* {@link GroupByKey} followed by {@code groupedValues(...)}.
*/
public static <K, V> GroupedValues<K, V, V> groupedValues(
SerializableFunction<Iterable<V>, V> fn) {
return groupedValues(IterableCombineFn.of(fn), displayDataForFn(fn));
}
/**
* Returns a {@link GroupedValues Combine.GroupedValues} {@code PTransform} that takes a {@code
* PCollection} of {@code KV}s where a key maps to an {@code Iterable} of values, e.g., the result
* of a {@code GroupByKey}, then uses the given {@code SerializableFunction} to combine all the
* values associated with a key, ignoring the key. The type of the input and output values must be
* the same.
*
* <p>Each output element has the same timestamp and is in the same window as its corresponding
* input element, and the output {@code PCollection} has the same {@link
* org.apache.beam.sdk.transforms.windowing.WindowFn} associated with it as the input.
*
* <p>See {@link GroupedValues Combine.GroupedValues} for more information.
*
* <p>Note that {@link #perKey(SerializableBiFunction)} is typically more convenient to use than
* {@link GroupByKey} followed by {@code groupedValues(...)}.
*/
public static <K, V> GroupedValues<K, V, V> groupedValues(SerializableBiFunction<V, V, V> fn) {
return groupedValues(BinaryCombineFn.of(fn), displayDataForFn(fn));
}
/**
* Returns a {@link GroupedValues Combine.GroupedValues} {@code PTransform} that takes a {@code
* PCollection} of {@code KV}s where a key maps to an {@code Iterable} of values, e.g., the result
* of a {@code GroupByKey}, then uses the given {@code CombineFn} to combine all the values
* associated with a key, ignoring the key. The types of the input and output values can differ.
*
* <p>Each output element has the same timestamp and is in the same window as its corresponding
* input element, and the output {@code PCollection} has the same {@link
* org.apache.beam.sdk.transforms.windowing.WindowFn} associated with it as the input.
*
* <p>See {@link GroupedValues Combine.GroupedValues} for more information.
*
* <p>Note that {@link #perKey(CombineFnBase.GlobalCombineFn)} is typically more convenient to use
* than {@link GroupByKey} followed by {@code groupedValues(...)}.
*/
public static <K, InputT, OutputT> GroupedValues<K, InputT, OutputT> groupedValues(
GlobalCombineFn<? super InputT, ?, OutputT> fn) {
return groupedValues(fn, displayDataForFn(fn));
}
private static <K, InputT, OutputT> GroupedValues<K, InputT, OutputT> groupedValues(
GlobalCombineFn<? super InputT, ?, OutputT> fn,
DisplayData.ItemSpec<? extends Class<?>> fnDisplayData) {
return new GroupedValues<>(fn, fnDisplayData);
}
/////////////////////////////////////////////////////////////////////////////
/**
* A {@code CombineFn<InputT, AccumT, OutputT>} specifies how to combine a collection of input
* values of type {@code InputT} into a single output value of type {@code OutputT}. It does this
* via one or more intermediate mutable accumulator values of type {@code AccumT}.
*
* <p>The overall process to combine a collection of input {@code InputT} values into a single
* output {@code OutputT} value is as follows:
*
* <ol>
* <li>The input {@code InputT} values are partitioned into one or more batches.
* <li>For each batch, the {@link #createAccumulator} operation is invoked to create a fresh
* mutable accumulator value of type {@code AccumT}, initialized to represent the
* combination of zero values.
* <li>For each input {@code InputT} value in a batch, the {@link #addInput} operation is
* invoked to add the value to that batch's accumulator {@code AccumT} value. The
* accumulator may just record the new value (e.g., if {@code AccumT == List<InputT>}, or
* may do work to represent the combination more compactly.
* <li>The {@link #mergeAccumulators} operation is invoked to combine a collection of
* accumulator {@code AccumT} values into a single combined output accumulator {@code
* AccumT} value, once the merging accumulators have had all all the input values in their
* batches added to them. This operation is invoked repeatedly, until there is only one
* accumulator value left.
* <li>The {@link #extractOutput} operation is invoked on the final accumulator {@code AccumT}
* value to get the output {@code OutputT} value.
* </ol>
*
* <p>For example:
*
* <pre><code>
* public class AverageFn extends{@literal CombineFn<Integer, AverageFn.Accum, Double>} {
* public static class Accum implements Serializable {
* int sum = 0;
* int count = 0;
*
* {@literal @Override}
* public boolean equals(Object other) {
* if (other == null) return false;
* if (other == this) return true;
* if (!(other instanceof Accum))return false;
*
*
* Accum o = (Accum)other;
* if (this.sum != o.sum || this.count != o.count) {
* return false;
* } else {
* return true;
* }
* }
* }
*
* public Accum createAccumulator() {
* return new Accum();
* }
*
* public Accum addInput(Accum accum, Integer input) {
* accum.sum += input;
* accum.count++;
* return accum;
* }
*
* public Accum{@literal mergeAccumulators(Iterable<Accum> accums)} {
* Accum merged = createAccumulator();
* for (Accum accum : accums) {
* merged.sum += accum.sum;
* merged.count += accum.count;
* }
* return merged;
* }
*
* public Double extractOutput(Accum accum) {
* return ((double) accum.sum) / accum.count;
* }
* }{@literal
* PCollection<Integer> pc = ...;
* PCollection<Double> average = pc.apply(Combine.globally(new AverageFn()));
* }</code></pre>
*
* <p>Combining functions used by {@link Combine.Globally}, {@link Combine.PerKey}, {@link
* Combine.GroupedValues}, and {@code PTransforms} derived from them should be <i>associative</i>
* and <i>commutative</i>. Associativity is required because input values are first broken up into
* subgroups before being combined, and their intermediate results further combined, in an
* arbitrary tree structure. Commutativity is required because any order of the input values is
* ignored when breaking up input values into groups.
*
* <h3>Note on Data Encoding</h3>
*
* <p>Some form of data encoding is required when using custom types in a CombineFn which do not
* have well-known coders. The sample code above uses a custom Accumulator which gets coder by
* implementing {@link java.io.Serializable}. By doing this, we are relying on the generic {@link
* org.apache.beam.sdk.coders.CoderProvider}, which is able to provide a coder for any {@link
* java.io.Serializable} if applicable. In cases where {@link java.io.Serializable} is not
* efficient, or inapplicable, in general there are two alternatives for encoding:
*
* <ul>
* <li>Default {@link org.apache.beam.sdk.coders.CoderRegistry}. For example, implement a coder
* class explicitly and use the {@code @DefaultCoder} tag. See the {@link
* org.apache.beam.sdk.coders.CoderRegistry} for the numerous ways in which to bind a type
* to a coder.
* <li>CombineFn specific way. While extending CombineFn, overwrite both {@link
* #getAccumulatorCoder} and {@link #getDefaultOutputCoder}.
* </ul>
*
* @param <InputT> type of input values
* @param <AccumT> type of mutable accumulator values
* @param <OutputT> type of output values
*/
public abstract static class CombineFn<InputT, AccumT, OutputT>
extends AbstractGlobalCombineFn<InputT, AccumT, OutputT> {
/**
* Returns a new, mutable accumulator value, representing the accumulation of zero input values.
*/
public abstract AccumT createAccumulator();
/**
* Adds the given input value to the given accumulator, returning the new accumulator value.
*
* @param mutableAccumulator may be modified and returned for efficiency
* @param input should not be mutated
*/
public abstract AccumT addInput(AccumT mutableAccumulator, InputT input);
/**
* Returns an accumulator representing the accumulation of all the input values accumulated in
* the merging accumulators.
*
* @param accumulators only the first accumulator may be modified and returned for efficiency;
* the other accumulators should not be mutated, because they may be shared with other code
* and mutating them could lead to incorrect results or data corruption.
*/
public abstract AccumT mergeAccumulators(Iterable<AccumT> accumulators);
/**
* Returns the output value that is the result of combining all the input values represented by
* the given accumulator.
*
* @param accumulator can be modified for efficiency
*/
public abstract OutputT extractOutput(AccumT accumulator);
/**
* Returns an accumulator that represents the same logical value as the input accumulator, but
* may have a more compact representation.
*
* <p>For most CombineFns this would be a no-op, but should be overridden by CombineFns that
* (for example) buffer up elements and combine them in batches.
*
* <p>For efficiency, the input accumulator may be modified and returned.
*
* <p>By default returns the original accumulator.
*/
public AccumT compact(AccumT accumulator) {
return accumulator;
}
/**
* Applies this {@code CombineFn} to a collection of input values to produce a combined output
* value.
*
* <p>Useful when using a {@code CombineFn} separately from a {@code Combine} transform. Does
* not invoke the {@link #mergeAccumulators} operation.
*/
public OutputT apply(Iterable<? extends InputT> inputs) {
AccumT accum = createAccumulator();
for (InputT input : inputs) {
accum = addInput(accum, input);
}
return extractOutput(accum);
}
/**
* {@inheritDoc}
*
* <p>By default returns the extract output of an empty accumulator.
*/
@Override
public OutputT defaultValue() {
return extractOutput(createAccumulator());
}
/**
* Returns a {@link TypeDescriptor} capturing what is known statically about the output type of
* this {@code CombineFn} instance's most-derived class.
*
* <p>In the normal case of a concrete {@code CombineFn} subclass with no generic type
* parameters of its own, this will be a complete non-generic type.
*/
public TypeDescriptor<OutputT> getOutputType() {
return new TypeDescriptor<OutputT>(getClass()) {};
}
/**
* Returns a {@link TypeDescriptor} capturing what is known statically about the input type of
* this {@code CombineFn} instance's most-derived class.
*
* <p>In the normal case of a concrete {@code CombineFn} subclass with no generic type
* parameters of its own, this will be a complete non-generic type.
*/
public TypeDescriptor<InputT> getInputType() {
return new TypeDescriptor<InputT>(getClass()) {};
}
}
/////////////////////////////////////////////////////////////////////////////
/**
* An abstract subclass of {@link CombineFn} for implementing combiners that are more easily
* expressed as binary operations.
*/
public abstract static class BinaryCombineFn<V> extends CombineFn<V, Holder<V>, V> {
/**
* Returns a {@code CombineFn} that uses the given {@code SerializableBiFunction} to combine
* values.
*/
public static <V> BinaryCombineFn<V> of(SerializableBiFunction<V, V, V> combiner) {
return new BinaryCombineFn<V>() {
@Override
public V apply(V left, V right) {
return combiner.apply(left, right);
}
};
}
/** Applies the binary operation to the two operands, returning the result. */
public abstract V apply(V left, V right);
/** Returns the value that should be used for the combine of the empty set. */
@Nullable
public V identity() {
return null;
}
@Override
public Holder<V> createAccumulator() {
return new Holder<>();
}
@Override
public Holder<V> addInput(Holder<V> accumulator, V input) {
if (accumulator.present) {
accumulator.set(apply(accumulator.value, input));
} else {
accumulator.set(input);
}
return accumulator;
}
@Override
public Holder<V> mergeAccumulators(Iterable<Holder<V>> accumulators) {
Iterator<Holder<V>> iter = accumulators.iterator();
if (!iter.hasNext()) {
return createAccumulator();
} else {
Holder<V> running = iter.next();
while (iter.hasNext()) {
Holder<V> accum = iter.next();
if (accum.present) {
if (running.present) {
running.set(apply(running.value, accum.value));
} else {
running.set(accum.value);
}
}
}
return running;
}
}
@Override
public V extractOutput(Holder<V> accumulator) {
if (accumulator.present) {
return accumulator.value;
} else {
return identity();
}
}
@Override
public Coder<Holder<V>> getAccumulatorCoder(CoderRegistry registry, Coder<V> inputCoder) {
return new HolderCoder<>(inputCoder);
}
@Override
public Coder<V> getDefaultOutputCoder(CoderRegistry registry, Coder<V> inputCoder) {
return inputCoder;
}
}
/**
* Holds a single value value of type {@code V} which may or may not be present.
*
* <p>Used only as a private accumulator class.
*/
public static class Holder<V> {
@Nullable private V value;
private boolean present;
private Holder() {}
private Holder(V value) {
set(value);
}
private void set(V value) {
this.present = true;
this.value = value;
}
@Override
public String toString() {
return "Combine.Holder(value=" + value + ", present=" + present + ")";
}
}
/** A {@link Coder} for a {@link Holder}. */
private static class HolderCoder<V> extends StructuredCoder<Holder<V>> {
private Coder<V> valueCoder;
public HolderCoder(Coder<V> valueCoder) {
this.valueCoder = valueCoder;
}
@Override
public void encode(Holder<V> accumulator, OutputStream outStream)
throws CoderException, IOException {
encode(accumulator, outStream, Context.NESTED);
}
@Override
public void encode(Holder<V> accumulator, OutputStream outStream, Context context)
throws CoderException, IOException {
if (accumulator.present) {
outStream.write(1);
valueCoder.encode(accumulator.value, outStream, context);
} else {
outStream.write(0);
}
}
@Override
public Holder<V> decode(InputStream inStream) throws CoderException, IOException {
return decode(inStream, Context.NESTED);
}
@Override
public Holder<V> decode(InputStream inStream, Context context)
throws CoderException, IOException {
if (inStream.read() == 1) {
return new Holder<>(valueCoder.decode(inStream, context));
} else {
return new Holder<>();
}
}
@Override
public List<? extends Coder<?>> getCoderArguments() {
return Collections.singletonList(valueCoder);
}
@Override
public void verifyDeterministic() throws NonDeterministicException {
valueCoder.verifyDeterministic();
}
}
/**
* An abstract subclass of {@link CombineFn} for implementing combiners that are more easily and
* efficiently expressed as binary operations on <code>int</code>s
*
* <p>It uses {@code int[0]} as the mutable accumulator.
*/
public abstract static class BinaryCombineIntegerFn extends CombineFn<Integer, int[], Integer> {
/** Applies the binary operation to the two operands, returning the result. */
public abstract int apply(int left, int right);
/**
* Returns the identity element of this operation, i.e. an element {@code e} such that {@code
* apply(e, x) == apply(x, e) == x} for all values of {@code x}.
*/
public abstract int identity();
@Override
public int[] createAccumulator() {
return wrap(identity());
}
@Override
public int[] addInput(int[] accumulator, Integer input) {
accumulator[0] = apply(accumulator[0], input);
return accumulator;
}
@Override
public int[] mergeAccumulators(Iterable<int[]> accumulators) {
Iterator<int[]> iter = accumulators.iterator();
if (!iter.hasNext()) {
return createAccumulator();
} else {
int[] running = iter.next();
while (iter.hasNext()) {
running[0] = apply(running[0], iter.next()[0]);
}
return running;
}
}
@Override
public Integer extractOutput(int[] accumulator) {
return accumulator[0];
}
@Override
public Coder<int[]> getAccumulatorCoder(CoderRegistry registry, Coder<Integer> inputCoder) {
return DelegateCoder.of(
inputCoder, new ToIntegerCodingFunction(), new FromIntegerCodingFunction());
}
@Override
public Coder<Integer> getDefaultOutputCoder(CoderRegistry registry, Coder<Integer> inputCoder) {
return inputCoder;
}
private static int[] wrap(int value) {
return new int[] {value};
}
private static final class ToIntegerCodingFunction
implements DelegateCoder.CodingFunction<int[], Integer> {
@Override
public Integer apply(int[] accumulator) {
return accumulator[0];
}
@Override
public boolean equals(Object o) {
return o instanceof ToIntegerCodingFunction;
}
@Override
public int hashCode() {
return this.getClass().hashCode();
}
}
private static final class FromIntegerCodingFunction
implements DelegateCoder.CodingFunction<Integer, int[]> {
@Override
public int[] apply(Integer value) {
return wrap(value);
}
@Override
public boolean equals(Object o) {
return o instanceof FromIntegerCodingFunction;
}
@Override
public int hashCode() {
return this.getClass().hashCode();
}
}
}
/**
* An abstract subclass of {@link CombineFn} for implementing combiners that are more easily and
* efficiently expressed as binary operations on <code>long</code>s.
*
* <p>It uses {@code long[0]} as the mutable accumulator.
*/
public abstract static class BinaryCombineLongFn extends CombineFn<Long, long[], Long> {
/** Applies the binary operation to the two operands, returning the result. */
public abstract long apply(long left, long right);
/**
* Returns the identity element of this operation, i.e. an element {@code e} such that {@code
* apply(e, x) == apply(x, e) == x} for all values of {@code x}.
*/
public abstract long identity();
@Override
public long[] createAccumulator() {
return wrap(identity());
}
@Override
public long[] addInput(long[] accumulator, Long input) {
accumulator[0] = apply(accumulator[0], input);
return accumulator;
}
@Override
public long[] mergeAccumulators(Iterable<long[]> accumulators) {
Iterator<long[]> iter = accumulators.iterator();
if (!iter.hasNext()) {
return createAccumulator();
} else {
long[] running = iter.next();
while (iter.hasNext()) {
running[0] = apply(running[0], iter.next()[0]);
}
return running;
}
}
@Override
public Long extractOutput(long[] accumulator) {
return accumulator[0];
}
@Override
public Coder<long[]> getAccumulatorCoder(CoderRegistry registry, Coder<Long> inputCoder) {
return DelegateCoder.of(inputCoder, new ToLongCodingFunction(), new FromLongCodingFunction());
}
@Override
public Coder<Long> getDefaultOutputCoder(CoderRegistry registry, Coder<Long> inputCoder) {
return inputCoder;
}
private static long[] wrap(long value) {
return new long[] {value};
}
private static final class ToLongCodingFunction
implements DelegateCoder.CodingFunction<long[], Long> {
@Override
public Long apply(long[] accumulator) {
return accumulator[0];
}
@Override
public boolean equals(Object o) {
return o instanceof ToLongCodingFunction;
}
@Override
public int hashCode() {
return this.getClass().hashCode();
}
}
private static final class FromLongCodingFunction
implements DelegateCoder.CodingFunction<Long, long[]> {
@Override
public long[] apply(Long value) {
return wrap(value);
}
@Override
public boolean equals(Object o) {
return o instanceof FromLongCodingFunction;
}
@Override
public int hashCode() {
return this.getClass().hashCode();
}
}
}
/**
* An abstract subclass of {@link CombineFn} for implementing combiners that are more easily and
* efficiently expressed as binary operations on <code>double</code>s.
*
* <p>It uses {@code double[0]} as the mutable accumulator.
*/
public abstract static class BinaryCombineDoubleFn extends CombineFn<Double, double[], Double> {
/** Applies the binary operation to the two operands, returning the result. */
public abstract double apply(double left, double right);
/**
* Returns the identity element of this operation, i.e. an element {@code e} such that {@code
* apply(e, x) == apply(x, e) == x} for all values of {@code x}.
*/
public abstract double identity();
@Override
public double[] createAccumulator() {
return wrap(identity());
}
@Override
public double[] addInput(double[] accumulator, Double input) {
accumulator[0] = apply(accumulator[0], input);
return accumulator;
}
@Override
public double[] mergeAccumulators(Iterable<double[]> accumulators) {
Iterator<double[]> iter = accumulators.iterator();
if (!iter.hasNext()) {
return createAccumulator();
} else {
double[] running = iter.next();
while (iter.hasNext()) {
running[0] = apply(running[0], iter.next()[0]);
}
return running;
}
}
@Override
public Double extractOutput(double[] accumulator) {
return accumulator[0];
}
@Override
public Coder<double[]> getAccumulatorCoder(CoderRegistry registry, Coder<Double> inputCoder) {
return DelegateCoder.of(
inputCoder, new ToDoubleCodingFunction(), new FromDoubleCodingFunction());
}
@Override
public Coder<Double> getDefaultOutputCoder(CoderRegistry registry, Coder<Double> inputCoder) {
return inputCoder;
}
private static double[] wrap(double value) {
return new double[] {value};
}
private static final class ToDoubleCodingFunction
implements DelegateCoder.CodingFunction<double[], Double> {
@Override
public Double apply(double[] accumulator) {
return accumulator[0];
}
@Override
public boolean equals(Object o) {
return o instanceof ToDoubleCodingFunction;
}
@Override
public int hashCode() {
return this.getClass().hashCode();
}
}
private static final class FromDoubleCodingFunction
implements DelegateCoder.CodingFunction<Double, double[]> {
@Override
public double[] apply(Double value) {
return wrap(value);
}
@Override
public boolean equals(Object o) {
return o instanceof FromDoubleCodingFunction;
}
@Override
public int hashCode() {
return this.getClass().hashCode();
}
}
}
/////////////////////////////////////////////////////////////////////////////
/**
* A {@code CombineFn} that uses a subclass of {@link AccumulatingCombineFn.Accumulator} as its
* accumulator type. By defining the operations of the {@code Accumulator} helper class, the
* operations of the enclosing {@code CombineFn} are automatically provided. This can reduce the
* code required to implement a {@code CombineFn}.
*
* <p>For example, the example from {@link CombineFn} above can be expressed using {@code
* AccumulatingCombineFn} more concisely as follows:
*
* <pre>{@code
* public class AverageFn
* extends AccumulatingCombineFn<Integer, AverageFn.Accum, Double> {
* public Accum createAccumulator() {
* return new Accum();
* }
* public class Accum
* extends AccumulatingCombineFn<Integer, AverageFn.Accum, Double>
* .Accumulator {
* private int sum = 0;
* private int count = 0;
* public void addInput(Integer input) {
* sum += input;
* count++;
* }
* public void mergeAccumulator(Accum other) {
* sum += other.sum;
* count += other.count;
* }
* public Double extractOutput() {
* return ((double) sum) / count;
* }
* }
* }
* PCollection<Integer> pc = ...;
* PCollection<Double> average = pc.apply(Combine.globally(new AverageFn()));
* }</pre>
*
* @param <InputT> type of input values
* @param <AccumT> type of mutable accumulator values
* @param <OutputT> type of output values
*/
public abstract static class AccumulatingCombineFn<
InputT,
AccumT extends AccumulatingCombineFn.Accumulator<InputT, AccumT, OutputT>,
OutputT>
extends CombineFn<InputT, AccumT, OutputT> {
/** The type of mutable accumulator values used by this {@code AccumulatingCombineFn}. */
public interface Accumulator<InputT, AccumT, OutputT> {
/** Adds the given input value to this accumulator, modifying this accumulator. */
void addInput(InputT input);
/** Adds the input values represented by the given accumulator into this accumulator. */
void mergeAccumulator(AccumT other);
/**