Skip to content

Commit

Permalink
log: generate digest and log it in slow log (#9662) (#10093)
Browse files Browse the repository at this point in the history
  • Loading branch information
lysu authored and ngaut committed Apr 11, 2019
1 parent e5cfa51 commit a4707e2
Show file tree
Hide file tree
Showing 13 changed files with 51 additions and 53 deletions.
1 change: 1 addition & 0 deletions domain/topn_slow_query.go
Original file line number Diff line number Diff line change
Expand Up @@ -222,4 +222,5 @@ type SlowQueryInfo struct {
TableIDs string
IndexIDs string
Internal bool
Digest string
}
7 changes: 5 additions & 2 deletions executor/adapter.go
Original file line number Diff line number Diff line change
Expand Up @@ -403,9 +403,11 @@ func (a *ExecStmt) LogSlowQuery(txnTS uint64, succ bool) {
}
execDetail := sessVars.StmtCtx.GetExecDetails()
if costTime < threshold {
logutil.SlowQueryLogger.Debugf(sessVars.SlowLogFormat(txnTS, costTime, execDetail, indexIDs, sql))
_, digest := sessVars.StmtCtx.SQLDigest()
logutil.SlowQueryLogger.Debugf(sessVars.SlowLogFormat(txnTS, costTime, execDetail, indexIDs, digest, sql))
} else {
logutil.SlowQueryLogger.Warnf(sessVars.SlowLogFormat(txnTS, costTime, execDetail, indexIDs, sql))
_, digest := sessVars.StmtCtx.SQLDigest()
logutil.SlowQueryLogger.Warnf(sessVars.SlowLogFormat(txnTS, costTime, execDetail, indexIDs, digest, sql))
metrics.TotalQueryProcHistogram.Observe(costTime.Seconds())
metrics.TotalCopProcHistogram.Observe(execDetail.ProcessTime.Seconds())
metrics.TotalCopWaitHistogram.Observe(execDetail.WaitTime.Seconds())
Expand All @@ -415,6 +417,7 @@ func (a *ExecStmt) LogSlowQuery(txnTS uint64, succ bool) {
}
domain.GetDomain(a.Ctx).LogSlowQuery(&domain.SlowQueryInfo{
SQL: sql,
Digest: digest,
Start: a.StartTime,
Duration: costTime,
Detail: sessVars.StmtCtx.GetExecDetails(),
Expand Down
2 changes: 1 addition & 1 deletion executor/aggregate_test.go
Original file line number Diff line number Diff line change
Expand Up @@ -239,7 +239,7 @@ func (s *testSuite) TestAggregation(c *C) {

result = tk.MustQuery("select count(*) from information_schema.columns")
// When adding new memory columns in information_schema, please update this variable.
columnCountOfAllInformationSchemaTables := "784"
columnCountOfAllInformationSchemaTables := "785"
result.Check(testkit.Rows(columnCountOfAllInformationSchemaTables))

tk.MustExec("drop table if exists t1")
Expand Down
5 changes: 5 additions & 0 deletions executor/executor.go
Original file line number Diff line number Diff line change
Expand Up @@ -600,6 +600,7 @@ func (e *ShowSlowExec) Next(ctx context.Context, chk *chunk.Chunk) error {
} else {
chk.AppendInt64(11, 1)
}
chk.AppendString(12, slow.Digest)
e.cursor++
}
return nil
Expand Down Expand Up @@ -1359,6 +1360,10 @@ func ResetContextOfStmt(ctx sessionctx.Context, s ast.StmtNode) (err error) {
if err != nil {
return errors.Trace(err)
}
if s != nil {
// execute missed stmtID uses empty sql
sc.OriginalSQL = s.Text()
}
vars.StmtCtx = sc
return
}
1 change: 1 addition & 0 deletions executor/prepared.go
Original file line number Diff line number Diff line change
Expand Up @@ -271,6 +271,7 @@ func CompileExecutePreparedStmt(ctx sessionctx.Context, ID uint32, args ...inter
}
if prepared, ok := ctx.GetSessionVars().PreparedStmts[ID]; ok {
stmt.Text = prepared.Stmt.Text()
ctx.GetSessionVars().StmtCtx.OriginalSQL = stmt.Text
}
return stmt, nil
}
Expand Down
5 changes: 5 additions & 0 deletions infoschema/slow_log.go
Original file line number Diff line number Diff line change
Expand Up @@ -46,6 +46,7 @@ var slowQueryCols = []columnInfo{
{variable.SlowLogDBStr, mysql.TypeVarchar, 64, 0, nil, nil},
{variable.SlowLogIndexIDsStr, mysql.TypeVarchar, 100, 0, nil, nil},
{variable.SlowLogIsInternalStr, mysql.TypeTiny, 1, 0, nil, nil},
{variable.SlowLogDigestStr, mysql.TypeVarchar, 64, 0, nil, nil},
{variable.SlowLogQuerySQLStr, mysql.TypeVarchar, 4096, 0, nil, nil},
}

Expand Down Expand Up @@ -136,6 +137,7 @@ type slowQueryTuple struct {
db string
indexNames string
isInternal bool
digest string
sql string
}

Expand Down Expand Up @@ -212,6 +214,8 @@ func (st *slowQueryTuple) setFieldValue(tz *time.Location, field, value string)
st.indexNames = value
case variable.SlowLogIsInternalStr:
st.isInternal = value == "true"
case variable.SlowLogDigestStr:
st.digest = value
case variable.SlowLogQuerySQLStr:
st.sql = value
}
Expand All @@ -238,6 +242,7 @@ func (st *slowQueryTuple) convertToDatumRow() []types.Datum {
record = append(record, types.NewStringDatum(st.db))
record = append(record, types.NewStringDatum(st.indexNames))
record = append(record, types.NewDatum(st.isInternal))
record = append(record, types.NewStringDatum(st.digest))
record = append(record, types.NewStringDatum(st.sql))
return record
}
Expand Down
3 changes: 2 additions & 1 deletion infoschema/slow_log_test.go
Original file line number Diff line number Diff line change
Expand Up @@ -30,6 +30,7 @@ func (s *testSuite) TestParseSlowLogFile(c *C) {
# Query_time: 0.216905
# Process_time: 0.021 Request_count: 1 Total_keys: 637 Processed_keys: 436
# Is_internal: true
# Digest: 42a1c8aae6f133e934d4bf0147491709a8812ea05ff8819ec522780fe657b772
select * from t;`)
scanner := bufio.NewScanner(slowLog)
loc, err := time.LoadLocation("Asia/Shanghai")
Expand All @@ -46,7 +47,7 @@ select * from t;`)
}
recordString += str
}
expectRecordString := "2019-01-24 22:32:29.313255,405888132465033227,,0,0.216905,0.021,0,0,1,637,0,,,1,select * from t;"
expectRecordString := "2019-01-24 22:32:29.313255,405888132465033227,,0,0.216905,0.021,0,0,1,637,0,,,1,42a1c8aae6f133e934d4bf0147491709a8812ea05ff8819ec522780fe657b772,select * from t;"
c.Assert(expectRecordString, Equals, recordString)
}

Expand Down
5 changes: 3 additions & 2 deletions infoschema/tables_test.go
Original file line number Diff line number Diff line change
Expand Up @@ -304,15 +304,16 @@ func (s *testSuite) TestSlowQuery(c *C) {
# Process_time: 0.161 Request_count: 1 Total_keys: 100001 Process_keys: 100000
# DB: test
# Is_internal: false
# Digest: 42a1c8aae6f133e934d4bf0147491709a8812ea05ff8819ec522780fe657b772
select * from t_slim;`))
c.Assert(f.Close(), IsNil)
c.Assert(err, IsNil)

tk.MustExec(fmt.Sprintf("set @@tidb_slow_query_file='%v'", slowLogFileName))
tk.MustExec("set time_zone = '+08:00';")
re := tk.MustQuery("select * from information_schema.slow_query")
re.Check(testutil.RowsWithSep("|", "2019-02-12 19:33:56.571953|406315658548871171|[email protected]|6|4.895492|0.161|0|0|1|100001|100000|test||0|select * from t_slim;"))
re.Check(testutil.RowsWithSep("|", "2019-02-12 19:33:56.571953|406315658548871171|[email protected]|6|4.895492|0.161|0|0|1|100001|100000|test||0|42a1c8aae6f133e934d4bf0147491709a8812ea05ff8819ec522780fe657b772|select * from t_slim;"))
tk.MustExec("set time_zone = '+00:00';")
re = tk.MustQuery("select * from information_schema.slow_query")
re.Check(testutil.RowsWithSep("|", "2019-02-12 11:33:56.571953|406315658548871171|[email protected]|6|4.895492|0.161|0|0|1|100001|100000|test||0|select * from t_slim;"))
re.Check(testutil.RowsWithSep("|", "2019-02-12 11:33:56.571953|406315658548871171|[email protected]|6|4.895492|0.161|0|0|1|100001|100000|test||0|42a1c8aae6f133e934d4bf0147491709a8812ea05ff8819ec522780fe657b772|select * from t_slim;"))
}
1 change: 1 addition & 0 deletions planner/core/planbuilder.go
Original file line number Diff line number Diff line change
Expand Up @@ -861,6 +861,7 @@ func buildShowSlowSchema() *expression.Schema {
schema.Append(buildColumn("", "TABLE_IDS", mysql.TypeVarchar, 256))
schema.Append(buildColumn("", "INDEX_IDS", mysql.TypeVarchar, 256))
schema.Append(buildColumn("", "INTERNAL", mysql.TypeTiny, tinySize))
schema.Append(buildColumn("", "DIGEST", mysql.TypeVarchar, 64))
return schema
}

Expand Down
16 changes: 16 additions & 0 deletions sessionctx/stmtctx/stmtctx.go
Original file line number Diff line number Diff line change
Expand Up @@ -18,6 +18,7 @@ import (
"sync"
"time"

"github.com/pingcap/parser"
"github.com/pingcap/parser/mysql"
"github.com/pingcap/tidb/util/execdetails"
"github.com/pingcap/tidb/util/memory"
Expand Down Expand Up @@ -93,6 +94,21 @@ type StatementContext struct {
IndexIDs []int64
StmtType string
Tables []TableEntry
OriginalSQL string
digestMemo struct {
sync.Once
normalized string
digest string
}
}

// SQLDigest gets normalized and digest for provided sql.
// it will cache result after first calling.
func (sc *StatementContext) SQLDigest() (normalized, sqlDigest string) {
sc.digestMemo.Do(func() {
sc.digestMemo.normalized, sc.digestMemo.digest = parser.NormalizeDigest(sc.OriginalSQL)
})
return sc.digestMemo.normalized, sc.digestMemo.digest
}

// TableEntry presents table in db.
Expand Down
7 changes: 6 additions & 1 deletion sessionctx/variable/session.go
Original file line number Diff line number Diff line change
Expand Up @@ -775,6 +775,8 @@ const (
SlowLogIsInternalStr = "Is_internal"
// SlowLogIndexIDsStr is slow log field name.
SlowLogIndexIDsStr = "Index_ids"
// SlowLogDigestStr is slow log field name.
SlowLogDigestStr = "Digest"
// SlowLogQuerySQLStr is slow log field name.
SlowLogQuerySQLStr = "Query" // use for slow log table, slow log will not print this field name but print sql directly.
)
Expand All @@ -791,7 +793,7 @@ const (
// # Index_ids: [1,2]
// # Is_internal: false
// select * from t_slim;
func (s *SessionVars) SlowLogFormat(txnTS uint64, costTime time.Duration, execDetail execdetails.ExecDetails, indexIDs string, sql string) string {
func (s *SessionVars) SlowLogFormat(txnTS uint64, costTime time.Duration, execDetail execdetails.ExecDetails, indexIDs string, digest, sql string) string {
var buf bytes.Buffer
execDetailStr := execDetail.String()
buf.WriteString(SlowLogPrefixStr + SlowLogTxnStartTSStr + SlowLogSpaceMarkStr + strconv.FormatUint(txnTS, 10) + "\n")
Expand All @@ -812,6 +814,9 @@ func (s *SessionVars) SlowLogFormat(txnTS uint64, costTime time.Duration, execDe
buf.WriteString(SlowLogPrefixStr + SlowLogIndexIDsStr + SlowLogSpaceMarkStr + indexIDs + "\n")
}
buf.WriteString(SlowLogPrefixStr + SlowLogIsInternalStr + SlowLogSpaceMarkStr + strconv.FormatBool(s.InRestrictedSQL) + "\n")
if len(digest) > 0 {
buf.WriteString(SlowLogPrefixStr + SlowLogDigestStr + SlowLogSpaceMarkStr + digest + "\n")
}
if len(sql) == 0 {
sql = ";"
}
Expand Down
6 changes: 5 additions & 1 deletion sessionctx/variable/session_test.go
Original file line number Diff line number Diff line change
Expand Up @@ -17,6 +17,7 @@ import (
"time"

. "github.com/pingcap/check"
"github.com/pingcap/parser"
"github.com/pingcap/parser/auth"
"github.com/pingcap/tidb/util/execdetails"
"github.com/pingcap/tidb/util/mock"
Expand Down Expand Up @@ -83,7 +84,10 @@ func (*testSessionSuite) TestSlowLogFormat(c *C) {
# DB: test
# Index_ids: [1,2]
# Is_internal: true
# Digest: 42a1c8aae6f133e934d4bf0147491709a8812ea05ff8819ec522780fe657b772
select * from t;`
logString := seVar.SlowLogFormat(txnTS, costTime, execDetail, "[1,2]", "select * from t")
sql := "select * from t"
digest := parser.DigestHash(sql)
logString := seVar.SlowLogFormat(txnTS, costTime, execDetail, "[1,2]", digest, sql)
c.Assert(logString, Equals, resultString)
}
45 changes: 0 additions & 45 deletions store/tikv/tikv_test.go
Original file line number Diff line number Diff line change
Expand Up @@ -14,14 +14,7 @@
package tikv

import (
"go/build"
"os"
"path"
"reflect"
"strings"

. "github.com/pingcap/check"
"github.com/pingcap/parser"
)

// OneByOneSuite is a suite, When with-tikv flag is true, there is only one storage, so the test suite have to run one by one.
Expand All @@ -45,41 +38,3 @@ type testTiKVSuite struct {
}

var _ = Suite(&testTiKVSuite{})

func getImportedPackages(c *C, srcDir string, pkgName string, pkgs *map[string][]string) {
if pkgName == "C" {
return
}
if _, exists := (*pkgs)[pkgName]; exists {
return
}
if strings.HasPrefix(pkgName, "golang_org") {
pkgName = path.Join("vendor", pkgName)
}
pkg, err := build.Import(pkgName, srcDir, 0)
c.Assert(err, IsNil)
(*pkgs)[pkgName] = pkg.Imports
for _, name := range (*pkgs)[pkgName] {
getImportedPackages(c, srcDir, name, pkgs)
}
}

// TestParserNoDep tests whether this package does not depend on tidb/parser.
func (s *testTiKVSuite) TestParserNoDep(c *C) {
srcDir, err := os.Getwd()
c.Assert(err, IsNil)

pkgs := make(map[string][]string)
currentPkgName := reflect.TypeOf(testTiKVSuite{}).PkgPath()
getImportedPackages(c, srcDir, currentPkgName, &pkgs)

parse := parser.New()
parserPkgName := reflect.TypeOf(*parse).PkgPath()

for pkgName, imports := range pkgs {
for _, importName := range imports {
c.Assert(importName == parserPkgName, IsFalse,
Commentf("`%s` is imported from `%s`, which is a child dependency of `%s`", parserPkgName, pkgName, currentPkgName))
}
}
}

0 comments on commit a4707e2

Please sign in to comment.