Skip to content

Commit 777382e

Browse files
committed
roachtest: add ternary logic partitioning (TLP) test
This commit adds a roachtest that performs ternary logic partitioning (TLP) testing. TLP is a method for logically testing a database which is based on the logical guarantee that for a given predicate `p`, all rows must satisfy exactly one of the following three predicates: `p`, `NOT p`, `p IS NULL`. Unioning the results of all three "partitions" should yield the same result as an "unpartitioned" query with a `true` predicate. TLP is implemented in [SQLancer](https://github.com/sqlancer/sqlancer) and more information can be found at https://www.manuelrigger.at/preprints/TLP.pdf. We currently implement a limited form of TLP that only runs queries of the form `SELECT * FROM table WHERE <predicate>` where `<predicate>` is randomly generated. We also only verify that the number of rows returned by the unpartitioned and partitioned queries are equal, not that the values of the rows are equal. See the documentation for `Smither.GenerateTLP` for more details. Release note: None
1 parent fd25b69 commit 777382e

File tree

6 files changed

+304
-0
lines changed

6 files changed

+304
-0
lines changed

pkg/cmd/roachtest/BUILD.bazel

Lines changed: 1 addition & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -111,6 +111,7 @@ go_library(
111111
"test.go",
112112
"test_registry.go",
113113
"test_runner.go",
114+
"tlp.go",
114115
"toxiproxy.go",
115116
"tpc_utils.go",
116117
"tpcc.go",

pkg/cmd/roachtest/registry.go

Lines changed: 1 addition & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -101,6 +101,7 @@ func registerTests(r *testRegistry) {
101101
registerSQLSmith(r)
102102
registerSyncTest(r)
103103
registerSysbench(r)
104+
registerTLP(r)
104105
registerTPCC(r)
105106
registerTPCDSVec(r)
106107
registerTPCE(r)

pkg/cmd/roachtest/tlp.go

Lines changed: 210 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -0,0 +1,210 @@
1+
// Copyright 2021 The Cockroach Authors.
2+
//
3+
// Use of this software is governed by the Business Source License
4+
// included in the file licenses/BSL.txt.
5+
//
6+
// As of the Change Date specified in that file, in accordance with
7+
// the Business Source License, use of this software will be governed
8+
// by the Apache License, Version 2.0, included in the file
9+
// licenses/APL.txt.
10+
package main
11+
12+
import (
13+
"context"
14+
gosql "database/sql"
15+
"fmt"
16+
"os"
17+
"path/filepath"
18+
"strings"
19+
"time"
20+
21+
"github.com/cockroachdb/cockroach/pkg/internal/sqlsmith"
22+
"github.com/cockroachdb/cockroach/pkg/util/randutil"
23+
"github.com/cockroachdb/errors"
24+
)
25+
26+
const statementTimeout = time.Minute
27+
28+
func registerTLP(r *testRegistry) {
29+
r.Add(testSpec{
30+
Name: "tlp",
31+
Owner: OwnerSQLQueries,
32+
Timeout: time.Minute * 5,
33+
MinVersion: "v20.2.0",
34+
Tags: nil,
35+
Cluster: makeClusterSpec(1),
36+
Run: runTLP,
37+
})
38+
}
39+
40+
func runTLP(ctx context.Context, t *test, c *cluster) {
41+
// Set up a statement logger for easy reproduction. We only
42+
// want to log successful statements and statements that
43+
// produced a TLP error.
44+
tlpLog, err := os.Create(filepath.Join(t.artifactsDir, "tlp.log"))
45+
if err != nil {
46+
t.Fatalf("could not create tlp.log: %v", err)
47+
}
48+
defer tlpLog.Close()
49+
logStmt := func(stmt string) {
50+
stmt = strings.TrimSpace(stmt)
51+
if stmt == "" {
52+
return
53+
}
54+
fmt.Fprint(tlpLog, stmt)
55+
if !strings.HasSuffix(stmt, ";") {
56+
fmt.Fprint(tlpLog, ";")
57+
}
58+
fmt.Fprint(tlpLog, "\n\n")
59+
}
60+
61+
conn := c.Conn(ctx, 1)
62+
63+
rnd, seed := randutil.NewPseudoRand()
64+
c.l.Printf("seed: %d", seed)
65+
66+
c.Put(ctx, cockroach, "./cockroach")
67+
if err := c.PutLibraries(ctx, "./lib"); err != nil {
68+
t.Fatalf("could not initialize libraries: %v", err)
69+
}
70+
c.Start(ctx, t)
71+
72+
setup := sqlsmith.Setups["rand-tables"](rnd)
73+
74+
t.Status("executing setup")
75+
c.l.Printf("setup:\n%s", setup)
76+
if _, err := conn.Exec(setup); err != nil {
77+
t.Fatal(err)
78+
} else {
79+
logStmt(setup)
80+
}
81+
82+
setStmtTimeout := fmt.Sprintf("SET statement_timeout='%s';", statementTimeout.String())
83+
t.Status("setting statement_timeout")
84+
c.l.Printf("statement timeout:\n%s", setStmtTimeout)
85+
if _, err := conn.Exec(setStmtTimeout); err != nil {
86+
t.Fatal(err)
87+
}
88+
logStmt(setStmtTimeout)
89+
90+
// Initialize a smither that generates only INSERT, UPDATE, and DELETE
91+
// statements with the MutationsOnly option. Smither.GenerateTLP always
92+
// returns SELECT queries, so the MutationsOnly option is used only for
93+
// randomly mutating the database.
94+
smither, err := sqlsmith.NewSmither(conn, rnd, sqlsmith.MutationsOnly())
95+
if err != nil {
96+
t.Fatal(err)
97+
}
98+
defer smither.Close()
99+
100+
t.Status("running TLP")
101+
until := time.After(t.spec.Timeout / 2)
102+
done := ctx.Done()
103+
for i := 1; ; i++ {
104+
select {
105+
case <-until:
106+
return
107+
case <-done:
108+
return
109+
default:
110+
}
111+
112+
if i%1000 == 0 {
113+
t.Status("running TLP: ", i, " statements completed")
114+
}
115+
116+
// Run 1000 mutations first so that the tables have rows. Run a mutation
117+
// for a tenth of the iterations after that to continually change the
118+
// state of the database.
119+
if i < 1000 || i%10 == 0 {
120+
runMutationStatement(conn, smither, logStmt)
121+
continue
122+
}
123+
124+
if err := runTLPQuery(conn, smither, logStmt); err != nil {
125+
t.Fatal(err)
126+
}
127+
}
128+
}
129+
130+
// runMutationsStatement runs a random INSERT, UPDATE, or DELETE statement that
131+
// potentially modifies the state of the database.
132+
func runMutationStatement(conn *gosql.DB, smither *sqlsmith.Smither, logStmt func(string)) {
133+
// Ignore panics from Generate.
134+
defer func() {
135+
if r := recover(); r != nil {
136+
return
137+
}
138+
}()
139+
140+
stmt := smither.Generate()
141+
142+
// Ignore timeouts.
143+
_ = runWithTimeout(func() error {
144+
// Ignore errors. Log successful statements.
145+
if _, err := conn.Exec(stmt); err == nil {
146+
logStmt(stmt)
147+
}
148+
return nil
149+
})
150+
}
151+
152+
// runTLPQuery runs two queries to perform TLP. If the results of the query are
153+
// not equal, an error is returned. Currently GenerateTLP always returns
154+
// unpartitioned and partitioned queries of the form "SELECT count(*) ...". The
155+
// resulting counts of the queries are compared in order to verify logical
156+
// correctness. See GenerateTLP for more information on TLP and the generated
157+
// queries.
158+
func runTLPQuery(conn *gosql.DB, smither *sqlsmith.Smither, logStmt func(string)) error {
159+
// Ignore panics from GenerateTLP.
160+
defer func() {
161+
if r := recover(); r != nil {
162+
return
163+
}
164+
}()
165+
166+
unpartitioned, partitioned := smither.GenerateTLP()
167+
168+
return runWithTimeout(func() error {
169+
var unpartitionedCount int
170+
row := conn.QueryRow(unpartitioned)
171+
if err := row.Scan(&unpartitionedCount); err != nil {
172+
// Ignore errors.
173+
//nolint:returnerrcheck
174+
return nil
175+
}
176+
177+
var partitionedCount int
178+
row = conn.QueryRow(partitioned)
179+
if err := row.Scan(&partitionedCount); err != nil {
180+
// Ignore errors.
181+
//nolint:returnerrcheck
182+
return nil
183+
}
184+
185+
if unpartitionedCount != partitionedCount {
186+
logStmt(unpartitioned)
187+
logStmt(partitioned)
188+
return errors.Newf(
189+
"expected unpartitioned count %d to equal partitioned count %d\nsql: %s\n%s",
190+
unpartitionedCount, partitionedCount, unpartitioned, partitioned)
191+
}
192+
193+
return nil
194+
})
195+
}
196+
197+
func runWithTimeout(f func() error) error {
198+
done := make(chan error, 1)
199+
go func() {
200+
err := f()
201+
done <- err
202+
}()
203+
select {
204+
case <-time.After(statementTimeout + time.Second*5):
205+
// Ignore timeouts.
206+
return nil
207+
case err := <-done:
208+
return err
209+
}
210+
}

pkg/internal/sqlsmith/BUILD.bazel

Lines changed: 1 addition & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -13,6 +13,7 @@ go_library(
1313
"scope.go",
1414
"setup.go",
1515
"sqlsmith.go",
16+
"tlp.go",
1617
"type.go",
1718
],
1819
importpath = "github.com/cockroachdb/cockroach/pkg/internal/sqlsmith",

pkg/internal/sqlsmith/sqlsmith.go

Lines changed: 10 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -292,6 +292,16 @@ var SimpleDatums = simpleOption("simple datums", func(s *Smither) {
292292
s.simpleDatums = true
293293
})
294294

295+
// MutationsOnly causes the Smither to emit 80% INSERT, 10% UPDATE, and 10%
296+
// DELETE statements.
297+
var MutationsOnly = simpleOption("mutations only", func(s *Smither) {
298+
s.stmtWeights = []statementWeight{
299+
{8, makeInsert},
300+
{1, makeUpdate},
301+
{1, makeDelete},
302+
}
303+
})
304+
295305
// IgnoreFNs causes the Smither to ignore functions that match the regex.
296306
func IgnoreFNs(regex string) SmitherOption {
297307
r := regexp.MustCompile(regex)

pkg/internal/sqlsmith/tlp.go

Lines changed: 81 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -0,0 +1,81 @@
1+
// Copyright 2021 The Cockroach Authors.
2+
//
3+
// Use of this software is governed by the Business Source License
4+
// included in the file licenses/BSL.txt.
5+
//
6+
// As of the Change Date specified in that file, in accordance with
7+
// the Business Source License, use of this software will be governed
8+
// by the Apache License, Version 2.0, included in the file
9+
// licenses/APL.txt.
10+
11+
package sqlsmith
12+
13+
import (
14+
"fmt"
15+
16+
"github.com/cockroachdb/cockroach/pkg/sql/sem/tree"
17+
"github.com/cockroachdb/errors"
18+
)
19+
20+
// GenerateTLP returns two SQL queries as strings that can be used for Ternary
21+
// Logic Partitioning (TLP). TLP is a method for logically testing DBMSs which
22+
// is based on the logical guarantee that for a given predicate p, all rows must
23+
// satisfy exactly one of the following three predicates: p, NOT p, p IS NULL.
24+
// TLP can find bugs when an unpartitioned query and a query partitioned into
25+
// three sub-queries do not yield the same results.
26+
//
27+
// More information on TLP: https://www.manuelrigger.at/preprints/TLP.pdf.
28+
//
29+
// We currently implement a limited form of TLP that can only verify that the
30+
// number of rows returned by the unpartitioned and the partitioned queries are
31+
// equal.
32+
//
33+
// This TLP implementation is also limited in the types of queries that are
34+
// tested. We currently only test basic SELECT query filters. It is possible to
35+
// use TLP to test aggregations, GROUP BY, HAVING, and JOINs, which have all
36+
// been implemented in SQLancer. See:
37+
// https://github.com/sqlancer/sqlancer/tree/1.1.0/src/sqlancer/cockroachdb/oracle/tlp.
38+
//
39+
// The first query returned is an unpartitioned query of the form:
40+
//
41+
// SELECT count(*) FROM table
42+
//
43+
// The second query returned is a partitioned query of the form:
44+
//
45+
// SELECT count(*) FROM (
46+
// SELECT * FROM table WHERE (p)
47+
// UNION ALL
48+
// SELECT * FROM table WHERE NOT (p)
49+
// UNION ALL
50+
// SELECT * FROM table WHERE (p) IS NULL
51+
// )
52+
//
53+
// If the resulting counts of the two queries are not equal, there is a logical
54+
// bug.
55+
func (s *Smither) GenerateTLP() (unpartitioned, partitioned string) {
56+
f := tree.NewFmtCtx(tree.FmtParsable)
57+
58+
table, _, _, cols, ok := s.getSchemaTable()
59+
if !ok {
60+
panic(errors.AssertionFailedf("failed to find random table"))
61+
}
62+
table.Format(f)
63+
tableName := f.CloseAndGetString()
64+
65+
unpartitioned = fmt.Sprintf("SELECT count(*) FROM %s", tableName)
66+
67+
pred := makeBoolExpr(s, cols)
68+
pred.Format(f)
69+
predicate := f.CloseAndGetString()
70+
71+
part1 := fmt.Sprintf("SELECT * FROM %s WHERE %s", tableName, predicate)
72+
part2 := fmt.Sprintf("SELECT * FROM %s WHERE NOT (%s)", tableName, predicate)
73+
part3 := fmt.Sprintf("SELECT * FROM %s WHERE (%s) IS NULL", tableName, predicate)
74+
75+
partitioned = fmt.Sprintf(
76+
"SELECT count(*) FROM (%s UNION ALL %s UNION ALL %s)",
77+
part1, part2, part3,
78+
)
79+
80+
return unpartitioned, partitioned
81+
}

0 commit comments

Comments
 (0)