Skip to content

Commit

Permalink
logutil, *: implement log desensitization (#3011) (#3266)
Browse files Browse the repository at this point in the history
* cherry pick #3011 to release-4.0

Signed-off-by: ti-srebot <ti-srebot@pingcap.com>

* fix conflict

Signed-off-by: Song Gao <disxiaofei@163.com>

* fix conflict

Signed-off-by: Song Gao <disxiaofei@163.com>

Co-authored-by: Song Gao <disxiaofei@163.com>
Co-authored-by: Ti Prow Robot <71242396+ti-community-prow-bot@users.noreply.github.com>
  • Loading branch information
3 people authored Dec 31, 2020
1 parent c3874d9 commit 7b36ffc
Show file tree
Hide file tree
Showing 9 changed files with 162 additions and 17 deletions.
66 changes: 66 additions & 0 deletions pkg/logutil/log.go
Original file line number Diff line number Diff line change
Expand Up @@ -21,6 +21,7 @@ import (
"runtime"
"strings"
"sync"
"sync/atomic"

"github.com/coreos/pkg/capnslog"
zaplog "github.com/pingcap/log"
Expand Down Expand Up @@ -292,3 +293,68 @@ func LogPanic() {
zaplog.Fatal("panic", zap.Reflect("recover", e))
}
}

var (
enabledRedactLog atomic.Value
)

func init() {
SetRedactLog(false)
}

// IsRedactLogEnabled indicates whether the log desensitization is enabled
func IsRedactLogEnabled() bool {
return enabledRedactLog.Load().(bool)
}

// SetRedactLog sets enabledRedactLog
func SetRedactLog(enabled bool) {
enabledRedactLog.Store(enabled)
}

// ZapRedactByteString receives []byte argument and return omitted information zap.Field if redact log enabled
func ZapRedactByteString(key string, arg []byte) zap.Field {
return zap.ByteString(key, RedactBytes(arg))
}

// ZapRedactString receives string argument and return omitted information in zap.Field if redact log enabled
func ZapRedactString(key, arg string) zap.Field {
return zap.String(key, RedactString(arg))
}

// ZapRedactStringer receives stringer argument and return omitted information in zap.Field if redact log enabled
func ZapRedactStringer(key string, arg fmt.Stringer) zap.Field {
return zap.Stringer(key, RedactStringer(arg))
}

// RedactBytes receives []byte argument and return omitted information if redact log enabled
func RedactBytes(arg []byte) []byte {
if IsRedactLogEnabled() {
return []byte("?")
}
return arg
}

// RedactString receives string argument and return omitted information if redact log enabled
func RedactString(arg string) string {
if IsRedactLogEnabled() {
return "?"
}
return arg
}

// RedactStringer receives stringer argument and return omitted information if redact log enabled
func RedactStringer(arg fmt.Stringer) fmt.Stringer {
if IsRedactLogEnabled() {
return stringer{}
}
return arg
}

type stringer struct {
}

// String implement fmt.Stringer
func (s stringer) String() string {
return "?"
}
53 changes: 53 additions & 0 deletions pkg/logutil/log_test.go
Original file line number Diff line number Diff line change
Expand Up @@ -15,6 +15,7 @@ package logutil

import (
"bytes"
"fmt"
"strings"
"testing"

Expand Down Expand Up @@ -109,3 +110,55 @@ func (s *testLogSuite) TestFileLog(c *C) {
c.Assert(InitFileLog(&zaplog.FileLogConfig{Filename: "/tmp"}), NotNil)
c.Assert(InitFileLog(&zaplog.FileLogConfig{Filename: "/tmp/test_file_log", MaxSize: 0}), IsNil)
}

func (s *testLogSuite) TestRedactLog(c *C) {
testcases := []struct {
name string
arg interface{}
enableRedactLog bool
expect interface{}
}{
{
name: "string arg, enable redact",
arg: "foo",
enableRedactLog: true,
expect: "?",
},
{
name: "string arg",
arg: "foo",
enableRedactLog: false,
expect: "foo",
},
{
name: "[]byte arg, enable redact",
arg: []byte("foo"),
enableRedactLog: true,
expect: []byte("?"),
},
{
name: "[]byte arg",
arg: []byte("foo"),
enableRedactLog: false,
expect: []byte("foo"),
},
}

for _, testcase := range testcases {
c.Log(testcase.name)
SetRedactLog(testcase.enableRedactLog)
switch testcase.arg.(type) {
case []byte:
r := RedactBytes(testcase.arg.([]byte))
c.Assert(r, DeepEquals, testcase.expect)
case string:
r := RedactString(testcase.arg.(string))
c.Assert(r, DeepEquals, testcase.expect)
case fmt.Stringer:
r := RedactStringer(testcase.arg.(fmt.Stringer))
c.Assert(r, DeepEquals, testcase.expect)
default:
panic("unmatched case")
}
}
}
9 changes: 4 additions & 5 deletions server/cluster/cluster.go
Original file line number Diff line number Diff line change
Expand Up @@ -532,16 +532,15 @@ func (c *RaftCluster) processRegionHeartbeat(region *core.RegionInfo) error {
if origin == nil {
log.Debug("insert new region",
zap.Uint64("region-id", region.GetID()),
zap.Stringer("meta-region", core.RegionToHexMeta(region.GetMeta())),
)
logutil.ZapRedactStringer("meta-region", core.RegionToHexMeta(region.GetMeta())))
saveKV, saveCache, isNew = true, true, true
} else {
r := region.GetRegionEpoch()
o := origin.GetRegionEpoch()
if r.GetVersion() > o.GetVersion() {
log.Info("region Version changed",
zap.Uint64("region-id", region.GetID()),
zap.String("detail", core.DiffRegionKeyInfo(origin, region)),
logutil.ZapRedactString("detail", core.DiffRegionKeyInfo(origin, region)),
zap.Uint64("old-version", o.GetVersion()),
zap.Uint64("new-version", r.GetVersion()),
)
Expand Down Expand Up @@ -621,7 +620,7 @@ func (c *RaftCluster) processRegionHeartbeat(region *core.RegionInfo) error {
if err := c.storage.DeleteRegion(item.GetMeta()); err != nil {
log.Error("failed to delete region from storage",
zap.Uint64("region-id", item.GetID()),
zap.Stringer("region-meta", core.RegionToHexMeta(item.GetMeta())),
logutil.ZapRedactStringer("region-meta", core.RegionToHexMeta(item.GetMeta())),
errs.ZapError(err))
}
}
Expand Down Expand Up @@ -673,7 +672,7 @@ func (c *RaftCluster) processRegionHeartbeat(region *core.RegionInfo) error {
// after restart. Here we only log the error then go on updating cache.
log.Error("failed to save region to storage",
zap.Uint64("region-id", region.GetID()),
zap.Stringer("region-meta", core.RegionToHexMeta(region.GetMeta())),
logutil.ZapRedactStringer("region-meta", core.RegionToHexMeta(region.GetMeta())),
errs.ZapError(err))
}
regionEventCounter.WithLabelValues("update_kv").Inc()
Expand Down
16 changes: 12 additions & 4 deletions server/cluster/cluster_worker.go
Original file line number Diff line number Diff line change
Expand Up @@ -22,6 +22,7 @@ import (
"github.com/pingcap/kvproto/pkg/pdpb"
"github.com/pingcap/log"
"github.com/tikv/pd/pkg/errs"
"github.com/tikv/pd/pkg/logutil"
"github.com/tikv/pd/server/core"
"github.com/tikv/pd/server/schedule"
"go.uber.org/zap"
Expand All @@ -33,6 +34,13 @@ func (c *RaftCluster) HandleRegionHeartbeat(region *core.RegionInfo) error {
return err
}

// If the region peer count is 0, then we should not handle this.
if len(region.GetPeers()) == 0 {
log.Warn("invalid region, zero region peer count",
logutil.ZapRedactStringer("region-meta", core.RegionToHexMeta(region.GetMeta())))
return errors.Errorf("invalid region, zero region peer count: %v", logutil.RedactStringer(core.RegionToHexMeta(region.GetMeta())))
}

c.RLock()
co := c.coordinator
c.RUnlock()
Expand Down Expand Up @@ -80,7 +88,7 @@ func (c *RaftCluster) ValidRequestRegion(reqRegion *metapb.Region) error {
startKey := reqRegion.GetStartKey()
region := c.GetRegionByKey(startKey)
if region == nil {
return errors.Errorf("region not found, request region: %v", core.RegionToHexMeta(reqRegion))
return errors.Errorf("region not found, request region: %v", logutil.RedactStringer(core.RegionToHexMeta(reqRegion)))
}
// If the request epoch is less than current region epoch, then returns an error.
reqRegionEpoch := reqRegion.GetRegionEpoch()
Expand Down Expand Up @@ -183,8 +191,8 @@ func (c *RaftCluster) HandleReportSplit(request *pdpb.ReportSplitRequest) (*pdpb
err := c.checkSplitRegion(left, right)
if err != nil {
log.Warn("report split region is invalid",
zap.Stringer("left-region", core.RegionToHexMeta(left)),
zap.Stringer("right-region", core.RegionToHexMeta(right)),
logutil.ZapRedactStringer("left-region", core.RegionToHexMeta(left)),
logutil.ZapRedactStringer("right-region", core.RegionToHexMeta(right)),
errs.ZapError(err))
return nil, err
}
Expand All @@ -195,7 +203,7 @@ func (c *RaftCluster) HandleReportSplit(request *pdpb.ReportSplitRequest) (*pdpb
originRegion.StartKey = left.GetStartKey()
log.Info("region split, generate new region",
zap.Uint64("region-id", originRegion.GetId()),
zap.Stringer("region-meta", core.RegionToHexMeta(left)))
logutil.ZapRedactStringer("region-meta", core.RegionToHexMeta(left)))
return &pdpb.ReportSplitResponse{}, nil
}

Expand Down
9 changes: 9 additions & 0 deletions server/config/config.go
Original file line number Diff line number Diff line change
Expand Up @@ -28,6 +28,7 @@ import (

"github.com/tikv/pd/pkg/errs"
"github.com/tikv/pd/pkg/grpcutil"
"github.com/tikv/pd/pkg/logutil"
"github.com/tikv/pd/pkg/metricutil"
"github.com/tikv/pd/pkg/typeutil"
"github.com/tikv/pd/server/schedule"
Expand Down Expand Up @@ -142,6 +143,8 @@ type Config struct {
Dashboard DashboardConfig `toml:"dashboard" json:"dashboard"`

ReplicationMode ReplicationModeConfig `toml:"replication-mode" json:"replication-mode"`
// EnableRedactLog indicates that whether redact log, 0 is disable. 1 is enable.
EnableRedactLog bool `toml:"enable-redact-log" json:"enable-redact-log"`
}

// NewConfig creates a new config.
Expand Down Expand Up @@ -218,6 +221,7 @@ const (

defaultDRWaitStoreTimeout = time.Minute
defaultDRWaitSyncTimeout = time.Minute
defaultEnableRedactLog = false
)

var (
Expand Down Expand Up @@ -536,6 +540,10 @@ func (c *Config) Adjust(meta *toml.MetaData) error {

c.ReplicationMode.adjust(configMetaData.Child("replication-mode"))

if !configMetaData.IsDefined("enable-redact-log") {
c.EnableRedactLog = defaultEnableRedactLog
}

return nil
}

Expand Down Expand Up @@ -1073,6 +1081,7 @@ func (c *Config) SetupLogger() error {
}
c.logger = lg
c.logProps = p
logutil.SetRedactLog(c.EnableRedactLog)
return nil
}

Expand Down
1 change: 1 addition & 0 deletions server/config/config_test.go
Original file line number Diff line number Diff line change
Expand Up @@ -160,6 +160,7 @@ leader-schedule-limit = 0
c.Assert(cfg.PreVote, IsTrue)
c.Assert(cfg.Schedule.MaxMergeRegionKeys, Equals, uint64(defaultMaxMergeRegionKeys))
c.Assert(cfg.PDServerCfg.MetricStorage, Equals, "http://127.0.0.1:9090")
c.Assert(cfg.EnableRedactLog, Equals, defaultEnableRedactLog)

// Check undefined config fields
cfgData = `
Expand Down
10 changes: 6 additions & 4 deletions server/core/region_tree.go
Original file line number Diff line number Diff line change
Expand Up @@ -21,6 +21,7 @@ import (
"github.com/pingcap/log"
"github.com/tikv/pd/pkg/btree"
"github.com/tikv/pd/pkg/errs"
"github.com/tikv/pd/pkg/logutil"
"go.uber.org/zap"
)

Expand Down Expand Up @@ -95,8 +96,8 @@ func (t *regionTree) update(region *RegionInfo) []*RegionInfo {
for _, item := range overlaps {
log.Debug("overlapping region",
zap.Uint64("region-id", item.GetID()),
zap.Stringer("delete-region", RegionToHexMeta(item.GetMeta())),
zap.Stringer("update-region", RegionToHexMeta(region.GetMeta())))
logutil.ZapRedactStringer("delete-region", RegionToHexMeta(item.GetMeta())),
logutil.ZapRedactStringer("update-region", RegionToHexMeta(region.GetMeta())))
t.tree.Delete(&regionItem{item})
}

Expand Down Expand Up @@ -229,8 +230,9 @@ func (t *regionTree) RandomRegion(ranges []KeyRange) *RegionInfo {
if endIndex <= startIndex {
if len(endKey) > 0 && bytes.Compare(startKey, endKey) > 0 {
log.Error("wrong range keys",
zap.String("start-key", string(HexRegionKey(startKey))),
zap.String("end-key", string(HexRegionKey(endKey))), errs.ZapError(errs.ErrWrongRangeKeys))
logutil.ZapRedactString("start-key", string(HexRegionKey(startKey))),
logutil.ZapRedactString("end-key", string(HexRegionKey(endKey))),
errs.ZapError(errs.ErrWrongRangeKeys))
}
continue
}
Expand Down
9 changes: 7 additions & 2 deletions server/replication/replication_mode.go
Original file line number Diff line number Diff line change
Expand Up @@ -24,6 +24,7 @@ import (
pb "github.com/pingcap/kvproto/pkg/replication_modepb"
"github.com/pingcap/log"
"github.com/tikv/pd/pkg/errs"
"github.com/tikv/pd/pkg/logutil"
"github.com/tikv/pd/server/config"
"github.com/tikv/pd/server/core"
"github.com/tikv/pd/server/schedule/opt"
Expand Down Expand Up @@ -389,7 +390,8 @@ func (m *ModeManager) updateProgress() {
for len(m.drRecoverKey) > 0 || m.drRecoverCount == 0 {
regions := m.cluster.ScanRegions(m.drRecoverKey, nil, regionScanBatchSize)
if len(regions) == 0 {
log.Warn("scan empty regions", zap.ByteString("recover-key", m.drRecoverKey))
log.Warn("scan empty regions",
logutil.ZapRedactByteString("recover-key", m.drRecoverKey))
return
}
for i, r := range regions {
Expand Down Expand Up @@ -440,7 +442,10 @@ func (m *ModeManager) estimateProgress() float32 {

func (m *ModeManager) checkRegionRecover(region *core.RegionInfo, startKey []byte) bool {
if !bytes.Equal(startKey, region.GetStartKey()) {
log.Warn("found region gap", zap.ByteString("key", startKey), zap.ByteString("region-start-key", region.GetStartKey()), zap.Uint64("region-id", region.GetID()))
log.Warn("found region gap",
logutil.ZapRedactByteString("key", startKey),
logutil.ZapRedactByteString("region-start-key", region.GetStartKey()),
zap.Uint64("region-id", region.GetID()))
return false
}
return region.GetReplicationStatus().GetStateId() == m.drAutoSync.StateID &&
Expand Down
6 changes: 4 additions & 2 deletions server/schedule/checker/merge_checker.go
Original file line number Diff line number Diff line change
Expand Up @@ -22,11 +22,11 @@ import (
"github.com/tikv/pd/pkg/cache"
"github.com/tikv/pd/pkg/codec"
"github.com/tikv/pd/pkg/errs"
"github.com/tikv/pd/pkg/logutil"
"github.com/tikv/pd/server/core"
"github.com/tikv/pd/server/schedule/operator"
"github.com/tikv/pd/server/schedule/opt"
"github.com/tikv/pd/server/schedule/placement"
"go.uber.org/zap"
)

// MergeChecker ensures region to merge with adjacent region when size is small
Expand Down Expand Up @@ -119,7 +119,9 @@ func (m *MergeChecker) Check(region *core.RegionInfo) []*operator.Operator {
return nil
}

log.Debug("try to merge region", zap.Stringer("from", core.RegionToHexMeta(region.GetMeta())), zap.Stringer("to", core.RegionToHexMeta(target.GetMeta())))
log.Debug("try to merge region",
logutil.ZapRedactStringer("from", core.RegionToHexMeta(region.GetMeta())),
logutil.ZapRedactStringer("to", core.RegionToHexMeta(target.GetMeta())))
ops, err := operator.CreateMergeRegionOperator("merge-region", m.cluster, region, target, operator.OpMerge)
if err != nil {
log.Warn("create merge region operator failed", errs.ZapError(err))
Expand Down

0 comments on commit 7b36ffc

Please sign in to comment.