Skip to content

Commit

Permalink
Merge #34454
Browse files Browse the repository at this point in the history
34454: sql: turn on query cache r=andy-kimball a=RaduBerinde

Release note (performance improvement): A query plan cache saves a
portion of the planning time for frequent queries.

Some benchmarks with KV: https://docs.google.com/spreadsheets/d/1n818OsKTWsatbpXo5ddOJ53mL4hY-JhNJEuIlytilLc/edit#gid=0

While there is some improvement, the implicit prepare case is still nowhere close to prepare-once; I plan to benchmark and do more work on the prepare path. I think it's a good idea to enable the cache now though, so we get more time to weed out any problems. 

Co-authored-by: Radu Berinde <radu@cockroachlabs.com>
Co-authored-by: Andrew Kimball <andyk@cockroachlabs.com>
  • Loading branch information
3 people committed Feb 5, 2019
2 parents dc4c803 + 317a77d commit 72858b6
Show file tree
Hide file tree
Showing 6 changed files with 38 additions and 14 deletions.
2 changes: 1 addition & 1 deletion docs/generated/settings/settings.html
Original file line number Diff line number Diff line change
Expand Up @@ -82,7 +82,7 @@
<tr><td><code>sql.metrics.statement_details.plan_collection.period</code></td><td>duration</td><td><code>5m0s</code></td><td>the time until a new logical plan is collected</td></tr>
<tr><td><code>sql.metrics.statement_details.threshold</code></td><td>duration</td><td><code>0s</code></td><td>minimum execution time to cause statistics to be collected</td></tr>
<tr><td><code>sql.parallel_scans.enabled</code></td><td>boolean</td><td><code>true</code></td><td>parallelizes scanning different ranges when the maximum result size can be deduced</td></tr>
<tr><td><code>sql.query_cache.enabled</code></td><td>boolean</td><td><code>false</code></td><td>enable the query cache</td></tr>
<tr><td><code>sql.query_cache.enabled</code></td><td>boolean</td><td><code>true</code></td><td>enable the query cache</td></tr>
<tr><td><code>sql.stats.experimental_automatic_collection.enabled</code></td><td>boolean</td><td><code>true</code></td><td>experimental automatic statistics collection mode</td></tr>
<tr><td><code>sql.tablecache.lease.refresh_limit</code></td><td>integer</td><td><code>50</code></td><td>maximum number of tables to periodically refresh leases for</td></tr>
<tr><td><code>sql.trace.log_statement_execute</code></td><td>boolean</td><td><code>false</code></td><td>set to true to enable logging of executed statements</td></tr>
Expand Down
4 changes: 0 additions & 4 deletions pkg/sql/conn_executor.go
Original file line number Diff line number Diff line change
Expand Up @@ -642,10 +642,6 @@ var maxStmtStatReset = settings.RegisterNonNegativeDurationSetting(
time.Hour*2, // 2 x diagnosticReportFrequency
)

var queryCacheEnabled = settings.RegisterBoolSetting(
"sql.query_cache.enabled", "enable the query cache", false,
)

// PeriodicallyClearStmtStats runs a loop to ensure that sql stats are reset.
// Usually we expect those stats to be reset by diagnostics reporting, after it
// generates its reports. However if the diagnostics loop crashes and stops
Expand Down
2 changes: 2 additions & 0 deletions pkg/sql/conn_executor_internal_test.go
Original file line number Diff line number Diff line change
Expand Up @@ -26,6 +26,7 @@ import (
"github.com/cockroachdb/cockroach/pkg/sql/distsqlrun"
"github.com/cockroachdb/cockroach/pkg/sql/parser"
"github.com/cockroachdb/cockroach/pkg/sql/pgwire/pgwirebase"
"github.com/cockroachdb/cockroach/pkg/sql/querycache"
"github.com/cockroachdb/cockroach/pkg/testutils"
"github.com/cockroachdb/cockroach/pkg/util/hlc"
"github.com/cockroachdb/cockroach/pkg/util/leaktest"
Expand Down Expand Up @@ -283,6 +284,7 @@ func startConnExecutor(
dummyLivenessProvider{}, /* liveness */
nil, /* nodeDialer */
),
QueryCache: querycache.New(0),
TestingKnobs: &ExecutorTestingKnobs{},
}
pool := mon.MakeUnlimitedMonitor(
Expand Down
2 changes: 2 additions & 0 deletions pkg/sql/descriptor_mutation_test.go
Original file line number Diff line number Diff line change
Expand Up @@ -86,6 +86,7 @@ func (mt mutationTest) makeMutationsActive() {
}
}
mt.tableDesc.Mutations = nil
mt.tableDesc.Version++
if err := mt.tableDesc.ValidateTable(cluster.MakeTestingClusterSettings()); err != nil {
mt.Fatal(err)
}
Expand Down Expand Up @@ -141,6 +142,7 @@ func (mt mutationTest) writeMutation(m sqlbase.DescriptorMutation) {
}
}
mt.tableDesc.Mutations = append(mt.tableDesc.Mutations, m)
mt.tableDesc.Version++
if err := mt.tableDesc.ValidateTable(cluster.MakeTestingClusterSettings()); err != nil {
mt.Fatal(err)
}
Expand Down
29 changes: 22 additions & 7 deletions pkg/sql/plan_opt.go
Original file line number Diff line number Diff line change
Expand Up @@ -18,6 +18,7 @@ import (
"context"
"fmt"

"github.com/cockroachdb/cockroach/pkg/settings"
"github.com/cockroachdb/cockroach/pkg/sql/opt/exec/execbuilder"
"github.com/cockroachdb/cockroach/pkg/sql/opt/memo"
"github.com/cockroachdb/cockroach/pkg/sql/opt/optbuilder"
Expand All @@ -28,6 +29,10 @@ import (
"github.com/cockroachdb/cockroach/pkg/util/log"
)

var queryCacheEnabled = settings.RegisterBoolSetting(
"sql.query_cache.enabled", "enable the query cache", true,
)

// prepareUsingOptimizer builds a memo for a prepared statement and populates
// the following stmt.Prepared fields:
// - Columns
Expand Down Expand Up @@ -69,8 +74,12 @@ func (p *planner) prepareUsingOptimizer(
stmt.Prepared.Memo = cachedData.Memo
return opc.flags, false, nil
}
opc.log(ctx, "query cache hit but datasources don't match (prepare)")
opc.log(ctx, "query cache hit but memo is stale (prepare)")
}
} else if ok {
opc.log(ctx, "query cache hit but there is no prepare metadata")
} else {
opc.log(ctx, "query cache miss")
}
opc.flags.Set(planFlagOptCacheMiss)
}
Expand Down Expand Up @@ -101,13 +110,17 @@ func (p *planner) prepareUsingOptimizer(
if opc.allowMemoReuse {
stmt.Prepared.Memo = memo
if opc.useCache {
// execPrepare sets the PrepareMetadata.InferredTypes field after this
// point. However, once the PrepareMetadata goes into the cache, it
// can't be modified without causing race conditions. So make a copy of
// it now.
// TODO(radu): Determine if the extra object allocation is really
// necessary.
pm := stmt.Prepared.PrepareMetadata
cachedData := querycache.CachedData{
SQL: stmt.SQL,
Memo: memo,
// We rely on stmt.Prepared.PrepareMetadata not being subsequently modified.
// TODO(radu): this also holds on to the memory referenced by other
// PreparedStatement fields.
PrepareMetadata: &stmt.Prepared.PrepareMetadata,
SQL: stmt.SQL,
Memo: memo,
PrepareMetadata: &pm,
}
p.execCfg.QueryCache.Add(&p.queryCacheSession, &cachedData)
}
Expand Down Expand Up @@ -305,10 +318,12 @@ func (opc *optPlanningCtx) buildExecMemo(
return nil, false, err
} else if isStale {
prepared.Memo, isCorrelated, err = opc.buildReusableMemo(ctx)
opc.log(ctx, "rebuilding cached memo")
if err != nil {
return nil, isCorrelated, err
}
}
opc.log(ctx, "reusing cached memo")
memo, err := opc.reuseMemo(prepared.Memo)
return memo, false, err
}
Expand Down
13 changes: 11 additions & 2 deletions pkg/sql/plan_opt_test.go
Original file line number Diff line number Diff line change
Expand Up @@ -38,6 +38,8 @@ type queryCacheTestHelper struct {

conns []*gosql.Conn
runners []*sqlutils.SQLRunner

hitsDelta, missesDelta int
}

func makeQueryCacheTestHelper(tb testing.TB, numConns int) *queryCacheTestHelper {
Expand All @@ -64,6 +66,7 @@ func makeQueryCacheTestHelper(tb testing.TB, numConns int) *queryCacheTestHelper
r.Exec(tb, "SET DATABASE = db1")
}
r0.Exec(tb, "SET CLUSTER SETTING sql.query_cache.enabled = true")
h.ResetStats()
return h
}

Expand All @@ -72,8 +75,14 @@ func (h *queryCacheTestHelper) Stop() {
}

func (h *queryCacheTestHelper) GetStats() (numHits, numMisses int) {
return int(h.srv.MustGetSQLCounter(MetaSQLOptPlanCacheHits.Name)),
int(h.srv.MustGetSQLCounter(MetaSQLOptPlanCacheMisses.Name))
return int(h.srv.MustGetSQLCounter(MetaSQLOptPlanCacheHits.Name)) - h.hitsDelta,
int(h.srv.MustGetSQLCounter(MetaSQLOptPlanCacheMisses.Name)) - h.missesDelta
}

func (h *queryCacheTestHelper) ResetStats() {
hits, misses := h.GetStats()
h.hitsDelta += hits
h.missesDelta += misses
}

func (h *queryCacheTestHelper) AssertStats(tb *testing.T, expHits, expMisses int) {
Expand Down

0 comments on commit 72858b6

Please sign in to comment.