Skip to content

Commit bde0f23

Browse files
authored
Merge pull request #156598 from andy-kimball/backport24.3-156595
release-24.3: sql: add read/write metrics
2 parents e209230 + 708f07c commit bde0f23

21 files changed

+367
-14
lines changed

docs/generated/metrics/metrics.html

Lines changed: 8 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -1779,6 +1779,14 @@
17791779
<tr><td>APPLICATION</td><td>sql.statements.active.internal</td><td>Number of currently active user SQL statements (internal queries)</td><td>SQL Internal Statements</td><td>GAUGE</td><td>COUNT</td><td>AVG</td><td>NONE</td></tr>
17801780
<tr><td>APPLICATION</td><td>sql.statements.auto_retry.count</td><td>Number of SQL statement automatic retries</td><td>SQL Statements</td><td>COUNTER</td><td>COUNT</td><td>AVG</td><td>NON_NEGATIVE_DERIVATIVE</td></tr>
17811781
<tr><td>APPLICATION</td><td>sql.statements.auto_retry.count.internal</td><td>Number of SQL statement automatic retries (internal queries)</td><td>SQL Internal Statements</td><td>COUNTER</td><td>COUNT</td><td>AVG</td><td>NON_NEGATIVE_DERIVATIVE</td></tr>
1782+
<tr><td>APPLICATION</td><td>sql.statements.bytes_read.count</td><td>Number of bytes read by SQL statements</td><td>Bytes</td><td>COUNTER</td><td>BYTES</td><td>AVG</td><td>NON_NEGATIVE_DERIVATIVE</td></tr>
1783+
<tr><td>APPLICATION</td><td>sql.statements.bytes_read.count.internal</td><td>Number of bytes read by SQL statements (internal queries)</td><td>SQL Internal Statements</td><td>COUNTER</td><td>BYTES</td><td>AVG</td><td>NON_NEGATIVE_DERIVATIVE</td></tr>
1784+
<tr><td>APPLICATION</td><td>sql.statements.index_bytes_written.count</td><td>Number of primary and secondary index bytes modified by SQL statements</td><td>Bytes</td><td>COUNTER</td><td>BYTES</td><td>AVG</td><td>NON_NEGATIVE_DERIVATIVE</td></tr>
1785+
<tr><td>APPLICATION</td><td>sql.statements.index_bytes_written.count.internal</td><td>Number of primary and secondary index bytes modified by SQL statements (internal queries)</td><td>SQL Internal Statements</td><td>COUNTER</td><td>BYTES</td><td>AVG</td><td>NON_NEGATIVE_DERIVATIVE</td></tr>
1786+
<tr><td>APPLICATION</td><td>sql.statements.index_rows_written.count</td><td>Number of primary and secondary index rows modified by SQL statements</td><td>Rows</td><td>COUNTER</td><td>COUNT</td><td>AVG</td><td>NON_NEGATIVE_DERIVATIVE</td></tr>
1787+
<tr><td>APPLICATION</td><td>sql.statements.index_rows_written.count.internal</td><td>Number of primary and secondary index rows modified by SQL statements (internal queries)</td><td>SQL Internal Statements</td><td>COUNTER</td><td>COUNT</td><td>AVG</td><td>NON_NEGATIVE_DERIVATIVE</td></tr>
1788+
<tr><td>APPLICATION</td><td>sql.statements.rows_read.count</td><td>Number of rows read by SQL statements</td><td>Rows</td><td>COUNTER</td><td>COUNT</td><td>AVG</td><td>NON_NEGATIVE_DERIVATIVE</td></tr>
1789+
<tr><td>APPLICATION</td><td>sql.statements.rows_read.count.internal</td><td>Number of rows read by SQL statements (internal queries)</td><td>SQL Internal Statements</td><td>COUNTER</td><td>COUNT</td><td>AVG</td><td>NON_NEGATIVE_DERIVATIVE</td></tr>
17821790
<tr><td>APPLICATION</td><td>sql.stats.activity.update.latency</td><td>The latency of updates made by the SQL activity updater job. Includes failed update attempts</td><td>Nanoseconds</td><td>HISTOGRAM</td><td>NANOSECONDS</td><td>AVG</td><td>NONE</td></tr>
17831791
<tr><td>APPLICATION</td><td>sql.stats.activity.updates.failed</td><td>Number of update attempts made by the SQL activity updater job that failed with errors</td><td>failed updates</td><td>COUNTER</td><td>COUNT</td><td>AVG</td><td>NON_NEGATIVE_DERIVATIVE</td></tr>
17841792
<tr><td>APPLICATION</td><td>sql.stats.activity.updates.successful</td><td>Number of successful updates made by the SQL activity updater job</td><td>successful updates</td><td>COUNTER</td><td>COUNT</td><td>AVG</td><td>NON_NEGATIVE_DERIVATIVE</td></tr>

pkg/roachprod/opentelemetry/cockroachdb_metrics.go

Lines changed: 8 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -1797,6 +1797,14 @@ var cockroachdbMetrics = map[string]string{
17971797
"sql_service_latency_sum": "sql.service.latency.sum",
17981798
"sql_statements_active": "sql.statements.active",
17991799
"sql_statements_active_internal": "sql.statements.active.internal",
1800+
"sql_statements_rows_read_count": "sql.statements.rows_read.count",
1801+
"sql_statements_rows_read_count_internal": "sql.statements.rows_read.count.internal",
1802+
"sql_statements_bytes_read_count": "sql.statements.bytes_read.count",
1803+
"sql_statements_bytes_read_count_internal": "sql.statements.bytes_read.count.internal",
1804+
"sql_statements_index_rows_written_count": "sql.statements.index_rows_written.count",
1805+
"sql_statements_index_rows_written_count_internal": "sql.statements.index_rows_written.count.internal",
1806+
"sql_statements_index_bytes_written_count": "sql.statements.index_bytes_written.count",
1807+
"sql_statements_index_bytes_written_count_internal": "sql.statements.index_bytes_written.count.internal",
18001808
"sql_stats_activity_update_latency": "sql.stats.activity.update.latency",
18011809
"sql_stats_activity_update_latency_bucket": "sql.stats.activity.update.latency.bucket",
18021810
"sql_stats_activity_update_latency_count": "sql.stats.activity.update.latency.count",

pkg/sql/BUILD.bazel

Lines changed: 1 addition & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -930,6 +930,7 @@ go_test(
930930
"//pkg/util/log/logconfig",
931931
"//pkg/util/log/logpb",
932932
"//pkg/util/log/logtestutils",
933+
"//pkg/util/metamorphic",
933934
"//pkg/util/metric",
934935
"//pkg/util/mon",
935936
"//pkg/util/protoutil",

pkg/sql/conn_executor.go

Lines changed: 4 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -551,6 +551,10 @@ func makeMetrics(internal bool) Metrics {
551551
FullTableOrIndexScanRejectedCount: metric.NewCounter(getMetricMeta(MetaFullTableOrIndexScanRejected, internal)),
552552
TxnRetryCount: metric.NewCounter(getMetricMeta(MetaTxnRetry, internal)),
553553
StatementRetryCount: metric.NewCounter(getMetricMeta(MetaStatementRetry, internal)),
554+
StatementRowsRead: metric.NewCounter(getMetricMeta(MetaStatementRowsRead, internal)),
555+
StatementBytesRead: metric.NewCounter(getMetricMeta(MetaStatementBytesRead, internal)),
556+
StatementIndexRowsWritten: metric.NewCounter(getMetricMeta(MetaStatementIndexRowsWritten, internal)),
557+
StatementIndexBytesWritten: metric.NewCounter(getMetricMeta(MetaStatementIndexBytesWritten, internal)),
554558
},
555559
StartedStatementCounters: makeStartedStatementCounters(internal),
556560
ExecutedStatementCounters: makeExecutedStatementCounters(internal),

pkg/sql/conn_executor_exec.go

Lines changed: 17 additions & 4 deletions
Original file line numberDiff line numberDiff line change
@@ -2405,12 +2405,23 @@ func (ex *connExecutor) makeExecPlan(
24052405

24062406
// topLevelQueryStats returns some basic statistics about the run of the query.
24072407
type topLevelQueryStats struct {
2408-
// bytesRead is the number of bytes read from disk.
2409-
bytesRead int64
2410-
// rowsRead is the number of rows read from disk.
2408+
// rowsRead is the number of rows read from primary and secondary indexes.
24112409
rowsRead int64
2412-
// rowsWritten is the number of rows written.
2410+
// bytesRead is the number of bytes read from primary and secondary indexes.
2411+
bytesRead int64
2412+
// rowsWritten is the number of rows written to the primary index. It does not
2413+
// include rows written to secondary indexes.
2414+
// NB: There is an asymmetry between rowsRead and rowsWritten - rowsRead
2415+
// includes rows read from secondary indexes, while rowsWritten does not
2416+
// include rows written to secondary indexes. This matches the behavior of
2417+
// EXPLAIN ANALYZE and SQL "rows affected".
24132418
rowsWritten int64
2419+
// indexRowsWritten is the number of rows written to primary and secondary
2420+
// indexes. It is always >= rowsWritten.
2421+
indexRowsWritten int64
2422+
// indexBytesWritten is the number of bytes written to primary and secondary
2423+
// indexes.
2424+
indexBytesWritten int64
24142425
// networkEgressEstimate is an estimate for the number of bytes sent to the
24152426
// client. It is used for estimating the number of RUs consumed by a query.
24162427
networkEgressEstimate int64
@@ -2424,6 +2435,8 @@ func (s *topLevelQueryStats) add(other *topLevelQueryStats) {
24242435
s.bytesRead += other.bytesRead
24252436
s.rowsRead += other.rowsRead
24262437
s.rowsWritten += other.rowsWritten
2438+
s.indexBytesWritten += other.indexBytesWritten
2439+
s.indexRowsWritten += other.indexRowsWritten
24272440
s.networkEgressEstimate += other.networkEgressEstimate
24282441
s.clientTime += other.clientTime
24292442
}

pkg/sql/delete.go

Lines changed: 9 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -232,6 +232,15 @@ func (d *deleteNode) rowsWritten() int64 {
232232
return d.run.td.rowsWritten
233233
}
234234

235+
func (d *deleteNode) indexRowsWritten() int64 {
236+
return d.run.td.indexRowsWritten
237+
}
238+
239+
func (d *deleteNode) indexBytesWritten() int64 {
240+
// No bytes counted as written for a deletion.
241+
return 0
242+
}
243+
235244
func (d *deleteNode) enableAutoCommit() {
236245
d.run.td.enableAutoCommit()
237246
}

pkg/sql/delete_range.go

Lines changed: 11 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -77,6 +77,17 @@ func (d *deleteRangeNode) rowsWritten() int64 {
7777
return int64(d.rowCount)
7878
}
7979

80+
func (d *deleteRangeNode) indexRowsWritten() int64 {
81+
// Same as rowsWritten, because deleteRangeNode only applies to primary index
82+
// rows (it is not used if there's a secondary index on the table).
83+
return int64(d.rowCount)
84+
}
85+
86+
func (d *deleteRangeNode) indexBytesWritten() int64 {
87+
// No bytes counted as written for a deletion.
88+
return 0
89+
}
90+
8091
// startExec implements the planNode interface.
8192
func (d *deleteRangeNode) startExec(params runParams) error {
8293
if err := params.p.cancelChecker.Check(); err != nil {

pkg/sql/distsql_running.go

Lines changed: 2 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -1451,6 +1451,8 @@ func (r *DistSQLReceiver) pushMeta(meta *execinfrapb.ProducerMetadata) execinfra
14511451
r.stats.bytesRead += meta.Metrics.BytesRead
14521452
r.stats.rowsRead += meta.Metrics.RowsRead
14531453
r.stats.rowsWritten += meta.Metrics.RowsWritten
1454+
r.stats.indexRowsWritten += meta.Metrics.IndexRowsWritten
1455+
r.stats.indexBytesWritten += meta.Metrics.IndexBytesWritten
14541456
if r.progressAtomic != nil && r.expectedRowsRead != 0 {
14551457
progress := float64(r.stats.rowsRead) / float64(r.expectedRowsRead)
14561458
atomic.StoreUint64(r.progressAtomic, math.Float64bits(progress))

pkg/sql/exec_util.go

Lines changed: 24 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -1204,6 +1204,30 @@ var (
12041204
Measurement: "SQL Statements",
12051205
Unit: metric.Unit_COUNT,
12061206
}
1207+
MetaStatementRowsRead = metric.Metadata{
1208+
Name: "sql.statements.rows_read.count",
1209+
Help: "Number of rows read by SQL statements",
1210+
Measurement: "Rows",
1211+
Unit: metric.Unit_COUNT,
1212+
}
1213+
MetaStatementBytesRead = metric.Metadata{
1214+
Name: "sql.statements.bytes_read.count",
1215+
Help: "Number of bytes read by SQL statements",
1216+
Measurement: "Bytes",
1217+
Unit: metric.Unit_BYTES,
1218+
}
1219+
MetaStatementIndexRowsWritten = metric.Metadata{
1220+
Name: "sql.statements.index_rows_written.count",
1221+
Help: "Number of primary and secondary index rows modified by SQL statements",
1222+
Measurement: "Rows",
1223+
Unit: metric.Unit_COUNT,
1224+
}
1225+
MetaStatementIndexBytesWritten = metric.Metadata{
1226+
Name: "sql.statements.index_bytes_written.count",
1227+
Help: "Number of primary and secondary index bytes modified by SQL statements",
1228+
Measurement: "Bytes",
1229+
Unit: metric.Unit_BYTES,
1230+
}
12071231
)
12081232

12091233
func getMetricMeta(meta metric.Metadata, internal bool) metric.Metadata {

pkg/sql/execinfrapb/data.proto

Lines changed: 14 additions & 3 deletions
Original file line numberDiff line numberDiff line change
@@ -334,12 +334,23 @@ message RemoteProducerMetadata {
334334
}
335335
// Metrics are unconditionally emitted by table readers.
336336
message Metrics {
337-
// Total number of bytes read while executing a statement.
337+
// BytesRead is the number of primary and secondary index bytes read while
338+
// executing a statement.
338339
optional int64 bytes_read = 1 [(gogoproto.nullable) = false];
339-
// Total number of rows read while executing a statement.
340+
// RowsRead is the number of primary and secondary index rows read while
341+
// executing a statement.
340342
optional int64 rows_read = 2 [(gogoproto.nullable) = false];
341-
// Total number of rows modified while executing a statement.
343+
// RowsWritten is the number of primary index rows modified while executing
344+
// a statement. It does not include secondary index rows.
342345
optional int64 rows_written = 3 [(gogoproto.nullable) = false];
346+
// IndexRowsWritten is the number of primary and secondary index rows
347+
// modified while executing a statement. It is always >= RowsWritten.
348+
optional int64 index_rows_written = 5 [(gogoproto.nullable) = false];
349+
// IndexBytesWritten is the number of primary and secondary index bytes
350+
// modified while executing a statement.
351+
optional int64 index_bytes_written = 6 [(gogoproto.nullable) = false];
352+
reserved 4;
353+
// NEXT ID: 7
343354
}
344355
oneof value {
345356
RangeInfos range_info = 1;

0 commit comments

Comments
 (0)