@@ -62,6 +62,7 @@ public class OperatorStats
62
62
private final Duration getOutputCpu ;
63
63
private final DataSize outputDataSize ;
64
64
private final long outputPositions ;
65
+ private final long updatedPositions ;
65
66
66
67
private final long dynamicFilterSplitsProcessed ;
67
68
private final Metrics metrics ;
@@ -117,6 +118,7 @@ public OperatorStats(
117
118
@ JsonProperty ("getOutputCpu" ) Duration getOutputCpu ,
118
119
@ JsonProperty ("outputDataSize" ) DataSize outputDataSize ,
119
120
@ JsonProperty ("outputPositions" ) long outputPositions ,
121
+ @ JsonProperty ("updatedPositions" ) long updatedPositions ,
120
122
121
123
@ JsonProperty ("dynamicFilterSplitsProcessed" ) long dynamicFilterSplitsProcessed ,
122
124
@ JsonProperty ("metrics" ) Metrics metrics ,
@@ -174,6 +176,8 @@ public OperatorStats(
174
176
this .outputDataSize = requireNonNull (outputDataSize , "outputDataSize is null" );
175
177
checkArgument (outputPositions >= 0 , "outputPositions is negative" );
176
178
this .outputPositions = outputPositions ;
179
+ checkArgument (updatedPositions >= 0 , "updatedPositions is negative" );
180
+ this .updatedPositions = updatedPositions ;
177
181
178
182
this .dynamicFilterSplitsProcessed = dynamicFilterSplitsProcessed ;
179
183
this .metrics = requireNonNull (metrics , "metrics is null" );
@@ -340,6 +344,12 @@ public long getOutputPositions()
340
344
return outputPositions ;
341
345
}
342
346
347
+ @ JsonProperty
348
+ public long getUpdatedPositions ()
349
+ {
350
+ return updatedPositions ;
351
+ }
352
+
343
353
@ JsonProperty
344
354
public long getDynamicFilterSplitsProcessed ()
345
355
{
@@ -485,6 +495,7 @@ private OperatorStats add(Iterable<OperatorStats> operators, Optional<Metrics> p
485
495
long getOutputCpu = this .getOutputCpu .roundTo (NANOSECONDS );
486
496
long outputDataSize = this .outputDataSize .toBytes ();
487
497
long outputPositions = this .outputPositions ;
498
+ long updatedPositions = this .updatedPositions ;
488
499
489
500
long dynamicFilterSplitsProcessed = this .dynamicFilterSplitsProcessed ;
490
501
Metrics .Accumulator metricsAccumulator = Metrics .accumulator ().add (this .getMetrics ());
@@ -535,6 +546,7 @@ private OperatorStats add(Iterable<OperatorStats> operators, Optional<Metrics> p
535
546
getOutputCpu += operator .getGetOutputCpu ().roundTo (NANOSECONDS );
536
547
outputDataSize += operator .getOutputDataSize ().toBytes ();
537
548
outputPositions += operator .getOutputPositions ();
549
+ updatedPositions += operator .getUpdatedPositions ();
538
550
539
551
dynamicFilterSplitsProcessed += operator .getDynamicFilterSplitsProcessed ();
540
552
metricsAccumulator .add (operator .getMetrics ());
@@ -596,6 +608,7 @@ private OperatorStats add(Iterable<OperatorStats> operators, Optional<Metrics> p
596
608
new Duration (getOutputCpu , NANOSECONDS ).convertToMostSuccinctTimeUnit (),
597
609
DataSize .ofBytes (outputDataSize ),
598
610
outputPositions ,
611
+ updatedPositions ,
599
612
600
613
dynamicFilterSplitsProcessed ,
601
614
metricsAccumulator .get (),
@@ -672,6 +685,7 @@ public OperatorStats summarize()
672
685
getOutputCpu ,
673
686
outputDataSize ,
674
687
outputPositions ,
688
+ updatedPositions ,
675
689
dynamicFilterSplitsProcessed ,
676
690
metrics ,
677
691
connectorMetrics ,
@@ -717,6 +731,7 @@ public OperatorStats withPipelineMetrics(Metrics pipelineMetrics)
717
731
getOutputCpu ,
718
732
outputDataSize ,
719
733
outputPositions ,
734
+ updatedPositions ,
720
735
dynamicFilterSplitsProcessed ,
721
736
metrics ,
722
737
connectorMetrics ,
0 commit comments