Skip to content

Commit a2de67e

Browse files
committed
sql: optimizer scan misestimate logging improvements
This commit is a follow-up to #154370. It changes misestimate logging to use structured logging on the `SQL_EXEC` log channel with a new `ScanRowCountMisestimate` event, instead of unstructured logs on DEV. It also adds an error case to `stats.EstimateStaleness()` when there are 1 or fewer full automatic stats collections, and uses the fully qualified table name in logs. Part of: #153748, #153873 Release note (sql change): Changes scan misestimate logging gated behind `sql.log.scan_row_count_misestimate.enabled` to use structured logging including the table and index being scanned, the estimated and actual row counts, the time since the last table stats collection, and the table's estimated staleness.
1 parent ed571f7 commit a2de67e

File tree

7 files changed

+184
-17
lines changed

7 files changed

+184
-17
lines changed

docs/generated/eventlog.md

Lines changed: 31 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -948,6 +948,37 @@ and the cluster setting `sql.log.all_statements.enabled` is set.
948948
| `BulkJobId` | The job id for bulk job (IMPORT/BACKUP/RESTORE). | no |
949949
| `StmtPosInTxn` | The statement's index in the transaction, starting at 1. | no |
950950

951+
### `scan_row_count_misestimate`
952+
953+
An event of type `scan_row_count_misestimate` is recorded when the optimizer's row count estimate
954+
for a logical scan differs significantly from the actual number of rows read,
955+
and cluster setting `sql.log.scan_row_count_misestimate.enabled` is set.
956+
957+
958+
| Field | Description | Sensitive |
959+
|--|--|--|
960+
| `TableName` | The fully qualified name of the table being scanned. | no |
961+
| `IndexName` | The name of the index being scanned. | no |
962+
| `EstimatedRowCount` | The optimizer's estimated row count for the scan. | no |
963+
| `ActualRowCount` | The actual number of rows read by all processors performing the scan. | no |
964+
| `NanosSinceStatsCollected` | Time in nanoseconds that have passed since full stats were collected on the table. | no |
965+
| `EstimatedStaleness` | Estimated fraction of stale rows in the table based on the time since stats were last collected. | no |
966+
967+
968+
#### Common fields
969+
970+
| Field | Description | Sensitive |
971+
|--|--|--|
972+
| `Timestamp` | The timestamp of the event. Expressed as nanoseconds since the Unix epoch. | no |
973+
| `EventType` | The type of the event. | no |
974+
| `Statement` | A normalized copy of the SQL statement that triggered the event. The statement string contains a mix of sensitive and non-sensitive details (it is redactable). | partially |
975+
| `Tag` | The statement tag. This is separate from the statement string, since the statement string can contain sensitive information. The tag is guaranteed not to. | no |
976+
| `User` | The user account that triggered the event. The special usernames `root` and `node` are not considered sensitive. | depends |
977+
| `DescriptorID` | The primary object descriptor affected by the operation. Set to zero for operations that don't affect descriptors. | no |
978+
| `ApplicationName` | The application name for the session where the event was emitted. This is included in the event to ease filtering of logging output by application. | no |
979+
| `PlaceholderValues` | The mapping of SQL placeholders to their values, for prepared statements. | yes |
980+
| `TxnReadTimestamp` | The current read timestamp of the transaction that triggered the event, if in a transaction. | no |
981+
951982
## SQL Logical Schema Changes
952983

953984
Events in this category pertain to DDL (Data Definition Language)

pkg/sql/distsql_running.go

Lines changed: 25 additions & 15 deletions
Original file line numberDiff line numberDiff line change
@@ -49,14 +49,14 @@ import (
4949
"github.com/cockroachdb/cockroach/pkg/sql/sessiondatapb"
5050
"github.com/cockroachdb/cockroach/pkg/sql/sqlerrors"
5151
"github.com/cockroachdb/cockroach/pkg/sql/sqltelemetry"
52-
"github.com/cockroachdb/cockroach/pkg/sql/stats"
5352
"github.com/cockroachdb/cockroach/pkg/sql/types"
5453
"github.com/cockroachdb/cockroach/pkg/util/buildutil"
5554
"github.com/cockroachdb/cockroach/pkg/util/errorutil/unimplemented"
5655
"github.com/cockroachdb/cockroach/pkg/util/hlc"
57-
"github.com/cockroachdb/cockroach/pkg/util/humanizeutil"
5856
"github.com/cockroachdb/cockroach/pkg/util/interval"
5957
"github.com/cockroachdb/cockroach/pkg/util/log"
58+
"github.com/cockroachdb/cockroach/pkg/util/log/eventpb"
59+
"github.com/cockroachdb/cockroach/pkg/util/log/severity"
6060
"github.com/cockroachdb/cockroach/pkg/util/mon"
6161
"github.com/cockroachdb/cockroach/pkg/util/quotapool"
6262
"github.com/cockroachdb/cockroach/pkg/util/ring"
@@ -1358,27 +1358,38 @@ type scanStageEstimate struct {
13581358
estimatedRowCount uint64
13591359
statsCreatedAt time.Time
13601360
tableID descpb.ID
1361-
tableName string
13621361
indexName string
13631362

13641363
rowsRead uint64
13651364
}
13661365

13671366
var misestimateLogLimiter = log.Every(10 * time.Second)
13681367

1369-
func (s *scanStageEstimate) logMisestimate(ctx context.Context, refresher *stats.Refresher) {
1370-
var suffix string
1368+
func (s *scanStageEstimate) logMisestimate(ctx context.Context, planner *planner) {
1369+
tn, err := planner.GetQualifiedTableNameByID(ctx, int64(s.tableID), tree.ResolveAnyTableKind)
1370+
if err != nil {
1371+
return
1372+
}
1373+
fqTableName := tn.FQString()
1374+
1375+
event := &eventpb.ScanRowCountMisestimate{
1376+
CommonSQLEventDetails: planner.getCommonSQLEventDetails(),
1377+
TableName: fqTableName,
1378+
IndexName: s.indexName,
1379+
EstimatedRowCount: s.estimatedRowCount,
1380+
ActualRowCount: s.rowsRead,
1381+
}
13711382
if !s.statsCreatedAt.IsZero() {
1372-
timeSinceStats := timeutil.Since(s.statsCreatedAt)
1373-
suffix = fmt.Sprintf("; table stats collected %s ago",
1374-
humanizeutil.LongDuration(timeSinceStats))
1375-
staleness, err := refresher.EstimateStaleness(ctx, s.tableID)
1376-
if err == nil {
1377-
suffix += fmt.Sprintf(" (estimated %.0f%% stale)", staleness*100)
1383+
nanosSinceStats := timeutil.Since(s.statsCreatedAt).Nanoseconds()
1384+
event.NanosSinceStatsCollected = nanosSinceStats
1385+
1386+
if estimatedStaleness, err :=
1387+
planner.ExecCfg().StatsRefresher.EstimateStaleness(ctx, s.tableID); err == nil {
1388+
event.EstimatedStaleness = estimatedStaleness
13781389
}
13791390
}
1380-
log.Dev.Warningf(ctx, "inaccurate estimate for scan on table %q (index %q): estimated=%d actual=%d%s",
1381-
s.tableName, s.indexName, s.estimatedRowCount, s.rowsRead, suffix)
1391+
1392+
log.StructuredEvent(ctx, severity.WARNING, event)
13821393
}
13831394

13841395
var _ execinfra.RowReceiver = &DistSQLReceiver{}
@@ -1844,7 +1855,6 @@ func (r *DistSQLReceiver) makeScanEstimates(physPlan *PhysicalPlan, st *cluster.
18441855
estimatedRowCount: p.Spec.EstimatedRowCount,
18451856
statsCreatedAt: p.Spec.StatsCreatedAt,
18461857
tableID: p.Spec.Core.TableReader.FetchSpec.TableID,
1847-
tableName: p.Spec.Core.TableReader.FetchSpec.TableName,
18481858
indexName: p.Spec.Core.TableReader.FetchSpec.IndexName,
18491859
}
18501860
}
@@ -1883,7 +1893,7 @@ func (r *DistSQLReceiver) maybeLogMisestimates(ctx context.Context, planner *pla
18831893
}
18841894
checkedLimiter = true
18851895
}
1886-
s.logMisestimate(ctx, planner.ExecCfg().StatsRefresher)
1896+
s.logMisestimate(ctx, planner)
18871897
}
18881898
}
18891899

pkg/sql/stats/automatic_stats.go

Lines changed: 3 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -936,6 +936,9 @@ func (r *Refresher) EstimateStaleness(ctx context.Context, tableID descpb.ID) (f
936936
staleTargetFraction := r.autoStatsFractionStaleRows(explicitSettings)
937937

938938
avgRefreshTime := avgFullRefreshTime(tableStats)
939+
if avgRefreshTime == defaultAverageTimeBetweenRefreshes {
940+
return 0, errors.New("insufficient auto stats history to estimate staleness")
941+
}
939942
statsAge := timeutil.Since(stat.CreatedAt)
940943
if r.knobs != nil && r.knobs.StubTimeNow != nil {
941944
statsAge = r.knobs.StubTimeNow().Sub(stat.CreatedAt)

pkg/sql/stats/automatic_stats_test.go

Lines changed: 28 additions & 2 deletions
Original file line numberDiff line numberDiff line change
@@ -9,6 +9,7 @@ import (
99
"context"
1010
"fmt"
1111
"math"
12+
"strings"
1213
"testing"
1314
"time"
1415

@@ -1146,7 +1147,7 @@ func TestEstimateStaleness(t *testing.T) {
11461147
return err
11471148
}
11481149

1149-
overwriteFullStats := func(startOffsetHours, intervalHours int) error {
1150+
overwriteFullStats := func(startOffsetHours, intervalHours, numStats int) error {
11501151
return s.DB().Txn(ctx, func(ctx context.Context, txn *kv.Txn) error {
11511152
_, err := internalDB.Executor().Exec(
11521153
ctx, "delete-stats", txn,
@@ -1157,7 +1158,7 @@ func TestEstimateStaleness(t *testing.T) {
11571158
return err
11581159
}
11591160

1160-
for i := 0; i < 5; i++ {
1161+
for i := 0; i < numStats; i++ {
11611162
columnIDsVal := tree.NewDArray(types.Int)
11621163
if err := columnIDsVal.Append(tree.NewDInt(tree.DInt(1))); err != nil {
11631164
return err
@@ -1190,10 +1191,33 @@ func TestEstimateStaleness(t *testing.T) {
11901191
require.Error(t, err)
11911192
require.Contains(t, err.Error(), "automatic stats collection is not allowed for this table")
11921193

1194+
// Ensure that we return an error if estimating staleness with insufficient
1195+
// auto stats history.
1196+
if err = overwriteFullStats(
1197+
5, /* startOffsetHours */
1198+
0, /* intervalHours */
1199+
1, /* numStats */
1200+
); err != nil {
1201+
t.Fatal(err)
1202+
}
1203+
1204+
err = testutils.SucceedsSoonError(func() error {
1205+
_, err := refresher.EstimateStaleness(ctx, table.GetID())
1206+
if err == nil {
1207+
return fmt.Errorf("expected error but got nil")
1208+
}
1209+
if !strings.Contains(err.Error(), "insufficient auto stats history to estimate staleness") {
1210+
return fmt.Errorf("expected 'insufficient auto stats history to estimate staleness' but got: %w", err)
1211+
}
1212+
return nil
1213+
})
1214+
require.NoError(t, err)
1215+
11931216
// Create stats with 10-hour intervals, the most recent being 5 hours old.
11941217
if err = overwriteFullStats(
11951218
5, /* startOffsetHours */
11961219
10, /* intervalHours */
1220+
5, /* numStats */
11971221
); err != nil {
11981222
t.Fatal(err)
11991223
}
@@ -1223,6 +1247,7 @@ func TestEstimateStaleness(t *testing.T) {
12231247
if err = overwriteFullStats(
12241248
15, /* startOffsetHours */
12251249
3, /* intervalHours */
1250+
5, /* numStats */
12261251
); err != nil {
12271252
t.Fatal(err)
12281253
}
@@ -1239,6 +1264,7 @@ func TestEstimateStaleness(t *testing.T) {
12391264
if err = overwriteFullStats(
12401265
15, /* startOffsetHours */
12411266
2, /* intervalHours */
1267+
5, /* numStats */
12421268
); err != nil {
12431269
t.Fatal(err)
12441270
}

pkg/util/log/eventpb/eventlog_channels_generated.go

Lines changed: 3 additions & 0 deletions
Some generated files are not rendered by default. Learn more about customizing how changed files appear on GitHub.

pkg/util/log/eventpb/json_encode_generated.go

Lines changed: 66 additions & 0 deletions
Some generated files are not rendered by default. Learn more about customizing how changed files appear on GitHub.

pkg/util/log/eventpb/sql_audit_events.proto

Lines changed: 28 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -269,3 +269,31 @@ message QueryExecute {
269269
CommonSQLEventDetails sql = 2 [(gogoproto.nullable) = false, (gogoproto.jsontag) = "", (gogoproto.embed) = true];
270270
CommonSQLExecDetails exec = 3 [(gogoproto.nullable) = false, (gogoproto.jsontag) = "", (gogoproto.embed) = true];
271271
}
272+
273+
// ScanRowCountMisestimate is recorded when the optimizer's row count estimate
274+
// for a logical scan differs significantly from the actual number of rows read,
275+
// and cluster setting `sql.log.scan_row_count_misestimate.enabled` is set.
276+
message ScanRowCountMisestimate {
277+
CommonEventDetails common = 1 [(gogoproto.nullable) = false, (gogoproto.jsontag) = "", (gogoproto.embed) = true];
278+
CommonSQLEventDetails sql = 2 [(gogoproto.nullable) = false, (gogoproto.jsontag) = "", (gogoproto.embed) = true];
279+
280+
// The fully qualified name of the table being scanned.
281+
string table_name = 3 [(gogoproto.jsontag) = ",omitempty", (gogoproto.moretags) = "redact:\"nonsensitive\""];
282+
283+
// The name of the index being scanned.
284+
string index_name = 4 [(gogoproto.jsontag) = ",omitempty", (gogoproto.moretags) = "redact:\"nonsensitive\""];
285+
286+
// The optimizer's estimated row count for the scan.
287+
uint64 estimated_row_count = 5 [(gogoproto.jsontag) = ",omitempty"];
288+
289+
// The actual number of rows read by all processors performing the scan.
290+
uint64 actual_row_count = 6 [(gogoproto.jsontag) = ",omitempty"];
291+
292+
// Time in nanoseconds that have passed since full stats were collected on
293+
// the table.
294+
int64 nanos_since_stats_collected = 7 [(gogoproto.jsontag) = ",omitempty"];
295+
296+
// Estimated fraction of stale rows in the table based on the time since stats
297+
// were last collected.
298+
double estimated_staleness = 8 [(gogoproto.jsontag) = ",omitempty"];
299+
}

0 commit comments

Comments
 (0)