implement new 'zrepl status'

Primary goals:

- Scrollable output ( fixes #245 )
- Sending job signals from status view
- Filtering of output by filesystem

Implementation:

- original TUI framework: github.com/rivo/tview
- but: tview is quasi-unmaintained, didn't support some features
- => use fork https://gitlab.com/tslocum/cview
- however, don't buy into either too much to avoid lock-in

- instead: **port over the existing status UI drawing code
  and adjust it to produce strings instead of directly
  drawing into the termbox buffer**

Co-authored-by: Calistoc <calistoc@protonmail.com>
Co-authored-by: InsanePrawn <insane.prawny@gmail.com>

fixes #245
fixes #220
This commit is contained in:
Christian Schwarz 2020-03-27 20:37:17 +01:00
parent 2c8c2cfa14
commit a58ce74ed0
17 changed files with 1712 additions and 817 deletions

View File

@ -1,816 +0,0 @@
package client
import (
"context"
"fmt"
"io"
"math"
"net/http"
"os"
"sort"
"strings"
"sync"
"time"
// tcell is the termbox-compatible library for abstracting away escape sequences, etc.
// as of tcell#252, the number of default distributed terminals is relatively limited
// additional terminal definitions can be included via side-effect import
// See https://github.com/gdamore/tcell/blob/master/terminfo/base/base.go
// See https://github.com/gdamore/tcell/issues/252#issuecomment-533836078
"github.com/gdamore/tcell/termbox"
_ "github.com/gdamore/tcell/terminfo/s/screen" // tmux on FreeBSD 11 & 12 without ncurses
"github.com/pkg/errors"
"github.com/spf13/pflag"
"github.com/zrepl/yaml-config"
"github.com/zrepl/zrepl/cli"
"github.com/zrepl/zrepl/daemon"
"github.com/zrepl/zrepl/daemon/job"
"github.com/zrepl/zrepl/daemon/pruner"
"github.com/zrepl/zrepl/daemon/snapper"
"github.com/zrepl/zrepl/replication/report"
)
type byteProgressMeasurement struct {
time time.Time
val int64
}
type bytesProgressHistory struct {
last *byteProgressMeasurement // pointer as poor man's optional
changeCount int
lastChange time.Time
bpsAvg float64
}
func (p *bytesProgressHistory) Update(currentVal int64) (bytesPerSecondAvg int64, changeCount int) {
if p.last == nil {
p.last = &byteProgressMeasurement{
time: time.Now(),
val: currentVal,
}
return 0, 0
}
if p.last.val != currentVal {
p.changeCount++
p.lastChange = time.Now()
}
if time.Since(p.lastChange) > 3*time.Second {
p.last = nil
return 0, 0
}
deltaV := currentVal - p.last.val
deltaT := time.Since(p.last.time)
rate := float64(deltaV) / deltaT.Seconds()
factor := 0.3
p.bpsAvg = (1-factor)*p.bpsAvg + factor*rate
p.last.time = time.Now()
p.last.val = currentVal
return int64(p.bpsAvg), p.changeCount
}
type tui struct {
x, y int
indent int
lock sync.Mutex //For report and error
report map[string]*job.Status
err error
jobFilter string
replicationProgress map[string]*bytesProgressHistory // by job name
}
func newTui() tui {
return tui{
replicationProgress: make(map[string]*bytesProgressHistory),
}
}
const INDENT_MULTIPLIER = 4
func (t *tui) moveLine(dl int, col int) {
t.y += dl
t.x = t.indent*INDENT_MULTIPLIER + col
}
func (t *tui) write(text string) {
for _, c := range text {
if c == '\n' {
t.newline()
continue
}
termbox.SetCell(t.x, t.y, c, termbox.ColorDefault, termbox.ColorDefault)
t.x += 1
}
}
func (t *tui) printf(text string, a ...interface{}) {
t.write(fmt.Sprintf(text, a...))
}
func wrap(s string, width int) string {
var b strings.Builder
for len(s) > 0 {
rem := width
if rem > len(s) {
rem = len(s)
}
if idx := strings.IndexAny(s, "\n\r"); idx != -1 && idx < rem {
rem = idx + 1
}
untilNewline := strings.TrimRight(s[:rem], "\n\r")
s = s[rem:]
if len(untilNewline) == 0 {
continue
}
b.WriteString(untilNewline)
b.WriteString("\n")
}
return strings.TrimRight(b.String(), "\n\r")
}
func (t *tui) printfDrawIndentedAndWrappedIfMultiline(format string, a ...interface{}) {
whole := fmt.Sprintf(format, a...)
width, _ := termbox.Size()
if !strings.ContainsAny(whole, "\n\r") && t.x+len(whole) <= width {
t.printf(format, a...)
} else {
t.addIndent(1)
t.newline()
t.write(wrap(whole, width-INDENT_MULTIPLIER*t.indent))
t.addIndent(-1)
}
}
func (t *tui) newline() {
t.moveLine(1, 0)
}
func (t *tui) setIndent(indent int) {
t.indent = indent
t.moveLine(0, 0)
}
func (t *tui) addIndent(indent int) {
t.indent += indent
t.moveLine(0, 0)
}
var statusFlags struct {
Raw bool
Job string
}
var StatusCmd = &cli.Subcommand{
Use: "status",
Short: "show job activity or dump as JSON for monitoring",
SetupFlags: func(f *pflag.FlagSet) {
f.BoolVar(&statusFlags.Raw, "raw", false, "dump raw status description from zrepl daemon")
f.StringVar(&statusFlags.Job, "job", "", "only dump specified job")
},
Run: runStatus,
}
func runStatus(ctx context.Context, s *cli.Subcommand, args []string) error {
httpc, err := controlHttpClient(s.Config().Global.Control.SockPath)
if err != nil {
return err
}
if statusFlags.Raw {
resp, err := httpc.Get("http://unix" + daemon.ControlJobEndpointStatus)
if err != nil {
return err
}
defer resp.Body.Close()
if resp.StatusCode != http.StatusOK {
fmt.Fprintf(os.Stderr, "Received error response:\n")
_, err := io.CopyN(os.Stderr, resp.Body, 4096)
if err != nil {
return err
}
return errors.Errorf("exit")
}
if _, err := io.Copy(os.Stdout, resp.Body); err != nil {
return err
}
return nil
}
t := newTui()
t.lock.Lock()
t.err = errors.New("Got no report yet")
t.lock.Unlock()
t.jobFilter = statusFlags.Job
err = termbox.Init()
if err != nil {
return err
}
defer termbox.Close()
update := func() {
var m daemon.Status
err2 := jsonRequestResponse(httpc, daemon.ControlJobEndpointStatus,
struct{}{},
&m,
)
t.lock.Lock()
t.err = err2
t.report = m.Jobs
t.lock.Unlock()
t.draw()
}
update()
ticker := time.NewTicker(500 * time.Millisecond)
defer ticker.Stop()
go func() {
for range ticker.C {
update()
}
}()
termbox.HideCursor()
termbox.Clear(termbox.ColorDefault, termbox.ColorDefault)
loop:
for {
switch ev := termbox.PollEvent(); ev.Type {
case termbox.EventKey:
switch ev.Key {
case termbox.KeyEsc:
break loop
case termbox.KeyCtrlC:
break loop
}
case termbox.EventResize:
t.draw()
}
}
return nil
}
func (t *tui) getReplicationProgressHistory(jobName string) *bytesProgressHistory {
p, ok := t.replicationProgress[jobName]
if !ok {
p = &bytesProgressHistory{}
t.replicationProgress[jobName] = p
}
return p
}
func (t *tui) draw() {
t.lock.Lock()
defer t.lock.Unlock()
termbox.Clear(termbox.ColorDefault, termbox.ColorDefault)
t.x = 0
t.y = 0
t.indent = 0
if t.err != nil {
t.write(t.err.Error())
} else {
//Iterate over map in alphabetical order
keys := make([]string, 0, len(t.report))
for k := range t.report {
if len(k) == 0 || daemon.IsInternalJobName(k) { //Internal job
continue
}
if t.jobFilter != "" && k != t.jobFilter {
continue
}
keys = append(keys, k)
}
sort.Strings(keys)
if len(keys) == 0 {
t.setIndent(0)
t.printf("no jobs to display")
t.newline()
termbox.Flush()
return
}
for _, k := range keys {
v := t.report[k]
t.setIndent(0)
t.printf("Job: %s", k)
t.setIndent(1)
t.newline()
t.printf("Type: %s", v.Type)
t.setIndent(1)
t.newline()
if v.Type == job.TypePush || v.Type == job.TypePull {
activeStatus, ok := v.JobSpecific.(*job.ActiveSideStatus)
if !ok || activeStatus == nil {
t.printf("ActiveSideStatus is null")
t.newline()
continue
}
t.printf("Replication:")
t.newline()
t.addIndent(1)
t.renderReplicationReport(activeStatus.Replication, t.getReplicationProgressHistory(k))
t.addIndent(-1)
t.printf("Pruning Sender:")
t.newline()
t.addIndent(1)
t.renderPrunerReport(activeStatus.PruningSender)
t.addIndent(-1)
t.printf("Pruning Receiver:")
t.newline()
t.addIndent(1)
t.renderPrunerReport(activeStatus.PruningReceiver)
t.addIndent(-1)
if v.Type == job.TypePush {
t.printf("Snapshotting:")
t.newline()
t.addIndent(1)
t.renderSnapperReport(activeStatus.Snapshotting)
t.addIndent(-1)
}
} else if v.Type == job.TypeSnap {
snapStatus, ok := v.JobSpecific.(*job.SnapJobStatus)
if !ok || snapStatus == nil {
t.printf("SnapJobStatus is null")
t.newline()
continue
}
t.printf("Pruning snapshots:")
t.newline()
t.addIndent(1)
t.renderPrunerReport(snapStatus.Pruning)
t.addIndent(-1)
t.printf("Snapshotting:")
t.newline()
t.addIndent(1)
t.renderSnapperReport(snapStatus.Snapshotting)
t.addIndent(-1)
} else if v.Type == job.TypeSource {
st := v.JobSpecific.(*job.PassiveStatus)
t.printf("Snapshotting:\n")
t.addIndent(1)
t.renderSnapperReport(st.Snapper)
t.addIndent(-1)
} else {
t.printf("No status representation for job type '%s', dumping as YAML", v.Type)
t.newline()
asYaml, err := yaml.Marshal(v.JobSpecific)
if err != nil {
t.printf("Error marshaling status to YAML: %s", err)
t.newline()
continue
}
t.write(string(asYaml))
t.newline()
continue
}
}
}
termbox.Flush()
}
func (t *tui) renderReplicationReport(rep *report.Report, history *bytesProgressHistory) {
if rep == nil {
t.printf("...\n")
return
}
if rep.WaitReconnectError != nil {
t.printfDrawIndentedAndWrappedIfMultiline("Connectivity: %s", rep.WaitReconnectError)
t.newline()
}
if !rep.WaitReconnectSince.IsZero() {
delta := time.Until(rep.WaitReconnectUntil).Round(time.Second)
if rep.WaitReconnectUntil.IsZero() || delta > 0 {
var until string
if rep.WaitReconnectUntil.IsZero() {
until = "waiting indefinitely"
} else {
until = fmt.Sprintf("hard fail in %s @ %s", delta, rep.WaitReconnectUntil)
}
t.printfDrawIndentedAndWrappedIfMultiline("Connectivity: reconnecting with exponential backoff (since %s) (%s)",
rep.WaitReconnectSince, until)
} else {
t.printfDrawIndentedAndWrappedIfMultiline("Connectivity: reconnects reached hard-fail timeout @ %s", rep.WaitReconnectUntil)
}
t.newline()
}
// TODO visualize more than the latest attempt by folding all attempts into one
if len(rep.Attempts) == 0 {
t.printf("no attempts made yet")
return
} else {
t.printf("Attempt #%d", len(rep.Attempts))
if len(rep.Attempts) > 1 {
t.printf(". Previous attempts failed with the following statuses:")
t.newline()
t.addIndent(1)
for i, a := range rep.Attempts[:len(rep.Attempts)-1] {
t.printfDrawIndentedAndWrappedIfMultiline("#%d: %s (failed at %s) (ran %s)", i+1, a.State, a.FinishAt, a.FinishAt.Sub(a.StartAt))
t.newline()
}
t.addIndent(-1)
} else {
t.newline()
}
}
latest := rep.Attempts[len(rep.Attempts)-1]
sort.Slice(latest.Filesystems, func(i, j int) bool {
return latest.Filesystems[i].Info.Name < latest.Filesystems[j].Info.Name
})
t.printf("Status: %s", latest.State)
t.newline()
if latest.State == report.AttemptPlanningError {
t.printf("Problem: ")
t.printfDrawIndentedAndWrappedIfMultiline("%s", latest.PlanError)
t.newline()
} else if latest.State == report.AttemptFanOutError {
t.printf("Problem: one or more of the filesystems encountered errors")
t.newline()
}
if latest.State != report.AttemptPlanning && latest.State != report.AttemptPlanningError {
// Draw global progress bar
// Progress: [---------------]
expected, replicated, containsInvalidSizeEstimates := latest.BytesSum()
rate, changeCount := history.Update(replicated)
eta := time.Duration(0)
if rate > 0 {
eta = time.Duration((expected-replicated)/rate) * time.Second
}
t.write("Progress: ")
t.drawBar(50, replicated, expected, changeCount)
t.write(fmt.Sprintf(" %s / %s @ %s/s", ByteCountBinary(replicated), ByteCountBinary(expected), ByteCountBinary(rate)))
if eta != 0 {
t.write(fmt.Sprintf(" (%s remaining)", humanizeDuration(eta)))
}
t.newline()
if containsInvalidSizeEstimates {
t.write("NOTE: not all steps could be size-estimated, total estimate is likely imprecise!")
t.newline()
}
if len(latest.Filesystems) == 0 {
t.write("NOTE: no filesystems were considered for replication!")
t.newline()
}
var maxFSLen int
for _, fs := range latest.Filesystems {
if len(fs.Info.Name) > maxFSLen {
maxFSLen = len(fs.Info.Name)
}
}
for _, fs := range latest.Filesystems {
t.printFilesystemStatus(fs, false, maxFSLen) // FIXME bring 'active' flag back
}
}
}
func (t *tui) renderPrunerReport(r *pruner.Report) {
if r == nil {
t.printf("...\n")
return
}
state, err := pruner.StateString(r.State)
if err != nil {
t.printf("Status: %q (parse error: %q)\n", r.State, err)
return
}
t.printf("Status: %s", state)
t.newline()
if r.Error != "" {
t.printf("Error: %s\n", r.Error)
}
type commonFS struct {
*pruner.FSReport
completed bool
}
all := make([]commonFS, 0, len(r.Pending)+len(r.Completed))
for i := range r.Pending {
all = append(all, commonFS{&r.Pending[i], false})
}
for i := range r.Completed {
all = append(all, commonFS{&r.Completed[i], true})
}
switch state {
case pruner.Plan:
fallthrough
case pruner.PlanErr:
return
}
if len(all) == 0 {
t.printf("nothing to do\n")
return
}
var totalDestroyCount, completedDestroyCount int
var maxFSname int
for _, fs := range all {
totalDestroyCount += len(fs.DestroyList)
if fs.completed {
completedDestroyCount += len(fs.DestroyList)
}
if maxFSname < len(fs.Filesystem) {
maxFSname = len(fs.Filesystem)
}
}
// global progress bar
progress := int(math.Round(80 * float64(completedDestroyCount) / float64(totalDestroyCount)))
t.write("Progress: ")
t.write("[")
t.write(times("=", progress))
t.write(">")
t.write(times("-", 80-progress))
t.write("]")
t.printf(" %d/%d snapshots", completedDestroyCount, totalDestroyCount)
t.newline()
sort.SliceStable(all, func(i, j int) bool {
return strings.Compare(all[i].Filesystem, all[j].Filesystem) == -1
})
// Draw a table-like representation of 'all'
for _, fs := range all {
t.write(rightPad(fs.Filesystem, maxFSname, " "))
t.write(" ")
if !fs.SkipReason.NotSkipped() {
t.printf("skipped: %s\n", fs.SkipReason)
continue
}
if fs.LastError != "" {
if strings.ContainsAny(fs.LastError, "\r\n") {
t.printf("ERROR:")
t.printfDrawIndentedAndWrappedIfMultiline("%s\n", fs.LastError)
} else {
t.printfDrawIndentedAndWrappedIfMultiline("ERROR: %s\n", fs.LastError)
}
t.newline()
continue
}
pruneRuleActionStr := fmt.Sprintf("(destroy %d of %d snapshots)",
len(fs.DestroyList), len(fs.SnapshotList))
if fs.completed {
t.printf("Completed %s\n", pruneRuleActionStr)
continue
}
t.write("Pending ") // whitespace is padding 10
if len(fs.DestroyList) == 1 {
t.write(fs.DestroyList[0].Name)
} else {
t.write(pruneRuleActionStr)
}
t.newline()
}
}
func (t *tui) renderSnapperReport(r *snapper.Report) {
if r == nil {
t.printf("<snapshot type does not have a report>\n")
return
}
t.printf("Status: %s", r.State)
t.newline()
if r.Error != "" {
t.printf("Error: %s\n", r.Error)
}
if !r.SleepUntil.IsZero() {
t.printf("Sleep until: %s\n", r.SleepUntil)
}
sort.Slice(r.Progress, func(i, j int) bool {
return strings.Compare(r.Progress[i].Path, r.Progress[j].Path) == -1
})
t.addIndent(1)
defer t.addIndent(-1)
dur := func(d time.Duration) string {
return d.Round(100 * time.Millisecond).String()
}
type row struct {
path, state, duration, remainder, hookReport string
}
var widths struct {
path, state, duration int
}
rows := make([]*row, len(r.Progress))
for i, fs := range r.Progress {
r := &row{
path: fs.Path,
state: fs.State.String(),
}
if fs.HooksHadError {
r.hookReport = fs.Hooks // FIXME render here, not in daemon
}
switch fs.State {
case snapper.SnapPending:
r.duration = "..."
r.remainder = ""
case snapper.SnapStarted:
r.duration = dur(time.Since(fs.StartAt))
r.remainder = fmt.Sprintf("snap name: %q", fs.SnapName)
case snapper.SnapDone:
fallthrough
case snapper.SnapError:
r.duration = dur(fs.DoneAt.Sub(fs.StartAt))
r.remainder = fmt.Sprintf("snap name: %q", fs.SnapName)
}
rows[i] = r
if len(r.path) > widths.path {
widths.path = len(r.path)
}
if len(r.state) > widths.state {
widths.state = len(r.state)
}
if len(r.duration) > widths.duration {
widths.duration = len(r.duration)
}
}
for _, r := range rows {
path := rightPad(r.path, widths.path, " ")
state := rightPad(r.state, widths.state, " ")
duration := rightPad(r.duration, widths.duration, " ")
t.printf("%s %s %s", path, state, duration)
t.printfDrawIndentedAndWrappedIfMultiline(" %s", r.remainder)
if r.hookReport != "" {
t.printfDrawIndentedAndWrappedIfMultiline("%s", r.hookReport)
}
t.newline()
}
}
func times(str string, n int) (out string) {
for i := 0; i < n; i++ {
out += str
}
return
}
func rightPad(str string, length int, pad string) string {
if len(str) > length {
return str[:length]
}
return str + strings.Repeat(pad, length-len(str))
}
var arrowPositions = `>\|/`
// changeCount = 0 indicates stall / no progress
func (t *tui) drawBar(length int, bytes, totalBytes int64, changeCount int) {
var completedLength int
if totalBytes > 0 {
completedLength = int(int64(length) * bytes / totalBytes)
if completedLength > length {
completedLength = length
}
} else if totalBytes == bytes {
completedLength = length
}
t.write("[")
t.write(times("=", completedLength))
t.write(string(arrowPositions[changeCount%len(arrowPositions)]))
t.write(times("-", length-completedLength))
t.write("]")
}
func (t *tui) printFilesystemStatus(rep *report.FilesystemReport, active bool, maxFS int) {
expected, replicated, containsInvalidSizeEstimates := rep.BytesSum()
sizeEstimationImpreciseNotice := ""
if containsInvalidSizeEstimates {
sizeEstimationImpreciseNotice = " (some steps lack size estimation)"
}
if rep.CurrentStep < len(rep.Steps) && rep.Steps[rep.CurrentStep].Info.BytesExpected == 0 {
sizeEstimationImpreciseNotice = " (step lacks size estimation)"
}
status := fmt.Sprintf("%s (step %d/%d, %s/%s)%s",
strings.ToUpper(string(rep.State)),
rep.CurrentStep, len(rep.Steps),
ByteCountBinary(replicated), ByteCountBinary(expected),
sizeEstimationImpreciseNotice,
)
activeIndicator := " "
if active {
activeIndicator = "*"
}
t.printf("%s %s %s ",
activeIndicator,
rightPad(rep.Info.Name, maxFS, " "),
status)
next := ""
if err := rep.Error(); err != nil {
next = err.Err
} else if rep.State != report.FilesystemDone {
if nextStep := rep.NextStep(); nextStep != nil {
if nextStep.IsIncremental() {
next = fmt.Sprintf("next: %s => %s", nextStep.Info.From, nextStep.Info.To)
} else {
next = fmt.Sprintf("next: full send %s", nextStep.Info.To)
}
attribs := []string{}
if nextStep.Info.Resumed {
attribs = append(attribs, "resumed")
}
attribs = append(attribs, fmt.Sprintf("encrypted=%s", nextStep.Info.Encrypted))
next += fmt.Sprintf(" (%s)", strings.Join(attribs, ", "))
} else {
next = "" // individual FSes may still be in planning state
}
}
t.printfDrawIndentedAndWrappedIfMultiline("%s", next)
t.newline()
}
func ByteCountBinary(b int64) string {
const unit = 1024
if b < unit {
return fmt.Sprintf("%d B", b)
}
div, exp := int64(unit), 0
for n := b / unit; n >= unit; n /= unit {
div *= unit
exp++
}
return fmt.Sprintf("%.1f %ciB", float64(b)/float64(div), "KMGTPE"[exp])
}
func humanizeDuration(duration time.Duration) string {
days := int64(duration.Hours() / 24)
hours := int64(math.Mod(duration.Hours(), 24))
minutes := int64(math.Mod(duration.Minutes(), 60))
seconds := int64(math.Mod(duration.Seconds(), 60))
var parts []string
force := false
chunks := []int64{days, hours, minutes, seconds}
for i, chunk := range chunks {
if force || chunk > 0 {
padding := 0
if force {
padding = 2
}
parts = append(parts, fmt.Sprintf("%*d%c", padding, chunk, "dhms"[i]))
force = true
}
}
return strings.Join(parts, " ")
}

View File

@ -0,0 +1,105 @@
package client
import (
"bytes"
"context"
"encoding/json"
"io"
"net"
"net/http"
"github.com/pkg/errors"
"github.com/zrepl/zrepl/daemon"
)
type Client struct {
h http.Client
}
func New(network, addr string) (*Client, error) {
httpc, err := controlHttpClient(func(_ context.Context) (net.Conn, error) { return net.Dial(network, addr) })
if err != nil {
return nil, err
}
return &Client{httpc}, nil
}
func (c *Client) Status() (s daemon.Status, _ error) {
err := jsonRequestResponse(c.h, daemon.ControlJobEndpointStatus,
struct{}{},
&s,
)
return s, err
}
func (c *Client) StatusRaw() ([]byte, error) {
var r json.RawMessage
err := jsonRequestResponse(c.h, daemon.ControlJobEndpointStatus, struct{}{}, &r)
if err != nil {
return nil, err
}
return r, nil
}
func (c *Client) signal(job, sig string) error {
return jsonRequestResponse(c.h, daemon.ControlJobEndpointSignal,
struct {
Name string
Op string
}{
Name: job,
Op: sig,
},
struct{}{},
)
}
func (c *Client) SignalReplication(job string) error {
return c.signal(job, "replication")
}
func (c *Client) SignalSnapshot(job string) error {
return c.signal(job, "snapshot")
}
func (c *Client) SignalReset(job string) error {
return c.signal(job, "reset")
}
func controlHttpClient(dialfunc func(context.Context) (net.Conn, error)) (client http.Client, err error) {
return http.Client{
Transport: &http.Transport{
DialContext: func(ctx context.Context, _, _ string) (net.Conn, error) {
return dialfunc(ctx)
},
},
}, nil
}
func jsonRequestResponse(c http.Client, endpoint string, req interface{}, res interface{}) error {
var buf bytes.Buffer
encodeErr := json.NewEncoder(&buf).Encode(req)
if encodeErr != nil {
return encodeErr
}
resp, err := c.Post("http://unix"+endpoint, "application/json", &buf)
if err != nil {
return err
}
defer resp.Body.Close()
if resp.StatusCode != http.StatusOK {
var msg bytes.Buffer
_, _ = io.CopyN(&msg, resp.Body, 4096) // ignore error, just display what we got
return errors.Errorf("%s", msg.String())
}
decodeError := json.NewDecoder(resp.Body).Decode(&res)
if decodeError != nil {
return decodeError
}
return nil
}

94
client/status/status.go Normal file
View File

@ -0,0 +1,94 @@
package status
import (
"context"
"os"
"time"
"github.com/mattn/go-isatty"
"github.com/pkg/errors"
"github.com/spf13/pflag"
"github.com/zrepl/zrepl/cli"
"github.com/zrepl/zrepl/client/status/client"
"github.com/zrepl/zrepl/config"
"github.com/zrepl/zrepl/daemon"
"github.com/zrepl/zrepl/util/choices"
)
type Client interface {
Status() (daemon.Status, error)
StatusRaw() ([]byte, error)
SignalReplication(job string) error
SignalSnapshot(job string) error
SignalReset(job string) error
}
type statusFlags struct {
Mode choices.Choices
Job string
Delay time.Duration
}
var statusv2Flags statusFlags
type statusv2Mode int
const (
StatusV2ModeInteractive statusv2Mode = 1 + iota
StatusV2ModeDump
StatusV2ModeRaw
StatusV2ModeLegacy
)
var Subcommand = &cli.Subcommand{
Use: "status",
Short: "retrieve & display daemon status information",
SetupFlags: func(f *pflag.FlagSet) {
statusv2Flags.Mode.Init(
"interactive", StatusV2ModeInteractive,
"dump", StatusV2ModeDump,
"raw", StatusV2ModeRaw,
"legacy", StatusV2ModeLegacy,
)
statusv2Flags.Mode.SetTypeString("mode")
statusv2Flags.Mode.SetDefaultValue(StatusV2ModeInteractive)
f.Var(&statusv2Flags.Mode, "mode", statusv2Flags.Mode.Usage())
f.StringVar(&statusv2Flags.Job, "job", "", "only show specified job (works in \"dump\" and \"interactive\" mode)")
f.DurationVarP(&statusv2Flags.Delay, "delay", "d", 1*time.Second, "use -d 3s for 3 seconds delay (minimum delay is 1s)")
},
Run: func(ctx context.Context, subcommand *cli.Subcommand, args []string) error {
return runStatusV2Command(ctx, subcommand.Config(), args)
},
}
func runStatusV2Command(ctx context.Context, config *config.Config, args []string) error {
c, err := client.New("unix", config.Global.Control.SockPath)
if err != nil {
return errors.Wrapf(err, "connect to daemon socket at %q", config.Global.Control.SockPath)
}
mode := statusv2Flags.Mode.Value().(statusv2Mode)
if !isatty.IsTerminal(os.Stdout.Fd()) && mode != StatusV2ModeDump {
usemode, err := statusv2Flags.Mode.InputForChoice(StatusV2ModeDump)
if err != nil {
panic(err)
}
return errors.Errorf("error: stdout is not a tty, please use --mode %s", usemode)
}
switch mode {
case StatusV2ModeInteractive:
return interactive(c, statusv2Flags)
case StatusV2ModeDump:
return dump(c, statusv2Flags.Job)
case StatusV2ModeRaw:
return raw(c)
case StatusV2ModeLegacy:
return legacy(c, statusv2Flags)
default:
panic("unreachable")
}
}

View File

@ -0,0 +1,70 @@
package status
import (
"fmt"
"os"
"strings"
"github.com/gdamore/tcell"
"github.com/mattn/go-isatty"
"github.com/pkg/errors"
"github.com/zrepl/zrepl/client/status/viewmodel"
)
func dump(c Client, job string) error {
s, err := c.Status()
if err != nil {
return err
}
if job != "" {
if _, ok := s.Jobs[job]; !ok {
return errors.Errorf("job %q not found", job)
}
}
width := (1 << 31) - 1
wrap := false
hline := strings.Repeat("-", 80)
if isatty.IsTerminal(os.Stdout.Fd()) {
wrap = true
screen, err := tcell.NewScreen()
if err != nil {
return errors.Wrap(err, "get terminal dimensions")
}
if err := screen.Init(); err != nil {
return errors.Wrap(err, "init screen")
}
width, _ = screen.Size()
screen.Fini()
hline = strings.Repeat("-", width)
}
m := viewmodel.New()
params := viewmodel.Params{
Report: s.Jobs,
ReportFetchError: nil,
SelectedJob: nil,
FSFilter: func(s string) bool { return true },
DetailViewWidth: width,
DetailViewWrap: wrap,
ShortKeybindingOverview: "",
}
m.Update(params)
for _, j := range m.Jobs() {
if job != "" && j.Name() != job {
continue
}
params.SelectedJob = j
m.Update(params)
fmt.Println(m.SelectedJob().FullDescription())
if job != "" {
return nil
} else {
fmt.Println(hline)
}
}
return nil
}

View File

@ -0,0 +1,347 @@
package status
import (
"fmt"
"regexp"
"strings"
"sync"
"time"
"github.com/gdamore/tcell/v2"
tview "gitlab.com/tslocum/cview"
"github.com/zrepl/zrepl/client/status/viewmodel"
)
func interactive(c Client, flag statusFlags) error {
// Set this so we don't overwrite the default terminal colors
// See https://github.com/rivo/tview/blob/master/styles.go
tview.Styles.PrimitiveBackgroundColor = tcell.ColorDefault
tview.Styles.ContrastBackgroundColor = tcell.ColorDefault
tview.Styles.PrimaryTextColor = tcell.ColorDefault
tview.Styles.BorderColor = tcell.ColorDefault
app := tview.NewApplication()
jobDetailSplit := tview.NewFlex()
jobMenu := tview.NewTreeView()
jobMenuRoot := tview.NewTreeNode("jobs")
jobMenuRoot.SetSelectable(true)
jobMenu.SetRoot(jobMenuRoot)
jobMenu.SetCurrentNode(jobMenuRoot)
jobMenu.SetSelectedTextColor(tcell.ColorGreen)
jobTextDetail := tview.NewTextView()
jobTextDetail.SetWrap(false)
jobMenu.SetBorder(true)
jobTextDetail.SetBorder(true)
toolbarSplit := tview.NewFlex()
toolbarSplit.SetDirection(tview.FlexRow)
inputBarContainer := tview.NewFlex()
fsFilterInput := tview.NewInputField()
fsFilterInput.SetBorder(false)
fsFilterInput.SetFieldBackgroundColor(tcell.ColorDefault)
inputBarLabel := tview.NewTextView()
inputBarLabel.SetText("[::b]FILTER ")
inputBarLabel.SetDynamicColors(true)
inputBarContainer.AddItem(inputBarLabel, 7, 1, false)
inputBarContainer.AddItem(fsFilterInput, 0, 10, false)
toolbarSplit.AddItem(inputBarContainer, 1, 0, false)
toolbarSplit.AddItem(jobDetailSplit, 0, 10, false)
bottombar := tview.NewFlex()
bottombar.SetDirection(tview.FlexColumn)
bottombarDateView := tview.NewTextView()
bottombar.AddItem(bottombarDateView, len(time.Now().String()), 0, false)
bottomBarStatus := tview.NewTextView()
bottomBarStatus.SetDynamicColors(true)
bottomBarStatus.SetTextAlign(tview.AlignRight)
bottombar.AddItem(bottomBarStatus, 0, 10, false)
toolbarSplit.AddItem(bottombar, 1, 0, false)
tabbableWithJobMenu := []tview.Primitive{jobMenu, jobTextDetail, fsFilterInput}
tabbableWithoutJobMenu := []tview.Primitive{jobTextDetail, fsFilterInput}
var tabbable []tview.Primitive
tabbableActiveIndex := 0
tabbableRedraw := func() {
if len(tabbable) == 0 {
app.SetFocus(nil)
return
}
if tabbableActiveIndex >= len(tabbable) {
app.SetFocus(tabbable[0])
return
}
app.SetFocus(tabbable[tabbableActiveIndex])
}
tabbableCycle := func() {
if len(tabbable) == 0 {
return
}
tabbableActiveIndex = (tabbableActiveIndex + 1) % len(tabbable)
app.SetFocus(tabbable[tabbableActiveIndex])
tabbableRedraw()
}
jobMenuVisisble := false
reconfigureJobDetailSplit := func(setJobMenuVisible bool) {
if jobMenuVisisble == setJobMenuVisible {
return
}
jobMenuVisisble = setJobMenuVisible
if setJobMenuVisible {
jobDetailSplit.RemoveItem(jobTextDetail)
jobDetailSplit.AddItem(jobMenu, 0, 1, true)
jobDetailSplit.AddItem(jobTextDetail, 0, 5, false)
tabbable = tabbableWithJobMenu
} else {
jobDetailSplit.RemoveItem(jobMenu)
tabbable = tabbableWithoutJobMenu
}
tabbableRedraw()
}
showModal := func(m *tview.Modal, modalDoneFunc func(idx int, label string)) {
preModalFocus := app.GetFocus()
m.SetDoneFunc(func(idx int, label string) {
if modalDoneFunc != nil {
modalDoneFunc(idx, label)
}
app.SetRoot(toolbarSplit, true)
app.SetFocus(preModalFocus)
app.Draw()
})
app.SetRoot(m, true)
app.Draw()
}
app.SetRoot(toolbarSplit, true)
// initial focus
tabbableActiveIndex = len(tabbable)
tabbableCycle()
reconfigureJobDetailSplit(true)
m := viewmodel.New()
params := &viewmodel.Params{
Report: nil,
SelectedJob: nil,
FSFilter: func(_ string) bool { return true },
DetailViewWidth: 100,
DetailViewWrap: false,
ShortKeybindingOverview: "[::b]Q[::-] quit [::b]<TAB>[::-] switch panes [::b]Shift+M[::-] toggle navbar [::b]Shift+S[::-] signal job [::b]</>[::-] filter filesystems",
}
paramsMtx := &sync.Mutex{}
var redraw func()
viewmodelupdate := func(cb func(*viewmodel.Params)) {
paramsMtx.Lock()
defer paramsMtx.Unlock()
cb(params)
m.Update(*params)
}
redraw = func() {
jobs := m.Jobs()
if flag.Job != "" {
job_found := false
for _, job := range jobs {
if strings.Compare(flag.Job, job.Name()) == 0 {
jobs = []*viewmodel.Job{job}
job_found = true
break
}
}
if !job_found {
jobs = nil
}
}
redrawJobsList := false
var selectedJobN *tview.TreeNode
if len(jobMenuRoot.GetChildren()) == len(jobs) {
for i, jobN := range jobMenuRoot.GetChildren() {
if jobN.GetReference().(*viewmodel.Job) != jobs[i] {
redrawJobsList = true
break
}
if jobN.GetReference().(*viewmodel.Job) == m.SelectedJob() {
selectedJobN = jobN
}
}
} else {
redrawJobsList = true
}
if redrawJobsList {
selectedJobN = nil
children := make([]*tview.TreeNode, len(jobs))
for i := range jobs {
jobN := tview.NewTreeNode(jobs[i].JobTreeTitle())
jobN.SetReference(jobs[i])
jobN.SetSelectable(true)
children[i] = jobN
jobN.SetSelectedFunc(func() {
viewmodelupdate(func(p *viewmodel.Params) {
p.SelectedJob = jobN.GetReference().(*viewmodel.Job)
})
})
if jobs[i] == m.SelectedJob() {
selectedJobN = jobN
}
}
jobMenuRoot.SetChildren(children)
}
if selectedJobN != nil && jobMenu.GetCurrentNode() != selectedJobN {
jobMenu.SetCurrentNode(selectedJobN)
} else if selectedJobN == nil {
// select something, otherwise selection breaks (likely bug in tview)
jobMenu.SetCurrentNode(jobMenuRoot)
}
if selJ := m.SelectedJob(); selJ != nil {
jobTextDetail.SetText(selJ.FullDescription())
} else {
jobTextDetail.SetText("please select a job")
}
bottombardatestring := m.DateString()
bottombarDateView.SetText(bottombardatestring)
bottombar.ResizeItem(bottombarDateView, len(bottombardatestring), 0)
bottomBarStatus.SetText(m.BottomBarStatus())
app.Draw()
}
go func() {
defer func() {
if err := recover(); err != nil {
app.Suspend(func() {
panic(err)
})
}
}()
for {
st, err := c.Status()
viewmodelupdate(func(p *viewmodel.Params) {
p.Report = st.Jobs
p.ReportFetchError = err
})
app.QueueUpdateDraw(redraw)
time.Sleep(flag.Delay)
}
}()
jobMenu.SetChangedFunc(func(jobN *tview.TreeNode) {
viewmodelupdate(func(p *viewmodel.Params) {
p.SelectedJob, _ = jobN.GetReference().(*viewmodel.Job)
})
redraw()
jobTextDetail.ScrollToBeginning()
})
jobMenu.SetSelectedFunc(func(jobN *tview.TreeNode) {
app.SetFocus(jobTextDetail)
})
app.SetBeforeDrawFunc(func(screen tcell.Screen) bool {
viewmodelupdate(func(p *viewmodel.Params) {
_, _, p.DetailViewWidth, _ = jobTextDetail.GetInnerRect()
})
return false
})
app.SetInputCapture(func(e *tcell.EventKey) *tcell.EventKey {
if e.Key() == tcell.KeyTab {
tabbableCycle()
return nil
}
if e.Key() == tcell.KeyRune && app.GetFocus() == fsFilterInput {
return e
}
if e.Key() == tcell.KeyRune && e.Rune() == '/' {
if app.GetFocus() != fsFilterInput {
app.SetFocus(fsFilterInput)
}
return e
}
if e.Key() == tcell.KeyRune && e.Rune() == 'M' {
reconfigureJobDetailSplit(!jobMenuVisisble)
return nil
}
if e.Key() == tcell.KeyRune && e.Rune() == 'q' {
app.Stop()
}
if e.Key() == tcell.KeyRune && e.Rune() == 'S' {
job, ok := jobMenu.GetCurrentNode().GetReference().(*viewmodel.Job)
if !ok {
return nil
}
signals := []string{"replication", "snapshot", "reset"}
clientFuncs := []func(job string) error{c.SignalReplication, c.SignalSnapshot, c.SignalReset}
sigMod := tview.NewModal()
sigMod.SetBackgroundColor(tcell.ColorDefault)
sigMod.SetBorder(true)
sigMod.GetForm().SetButtonTextColorFocused(tcell.ColorGreen)
sigMod.AddButtons(signals)
sigMod.SetText(fmt.Sprintf("Send a signal to job %q", job.Name()))
showModal(sigMod, func(idx int, _ string) {
go func() {
if idx == -1 {
return
}
err := clientFuncs[idx](job.Name())
if err != nil {
app.QueueUpdate(func() {
me := tview.NewModal()
me.SetText(fmt.Sprintf("signal error: %s", err))
me.AddButtons([]string{"Close"})
showModal(me, nil)
})
}
}()
})
}
return e
})
fsFilterInput.SetChangedFunc(func(searchterm string) {
viewmodelupdate(func(p *viewmodel.Params) {
p.FSFilter = func(fs string) bool {
r, err := regexp.Compile(searchterm)
if err != nil {
return true
}
return r.MatchString(fs)
}
})
redraw()
jobTextDetail.ScrollToBeginning()
})
fsFilterInput.SetInputCapture(func(event *tcell.EventKey) *tcell.EventKey {
if event.Key() == tcell.KeyEnter {
app.SetFocus(jobTextDetail)
return nil
}
return event
})
jobTextDetail.SetInputCapture(func(event *tcell.EventKey) *tcell.EventKey {
if event.Key() == tcell.KeyRune && event.Rune() == 'w' {
// toggle wrapping
viewmodelupdate(func(p *viewmodel.Params) {
p.DetailViewWrap = !p.DetailViewWrap
})
redraw()
return nil
}
return event
})
return app.Run()
}

View File

@ -0,0 +1,128 @@
package status
import (
"fmt"
"os"
"strings"
"sync"
"time"
"github.com/gdamore/tcell/v2"
"github.com/mattn/go-isatty"
"github.com/pkg/errors"
tview "gitlab.com/tslocum/cview"
"github.com/zrepl/zrepl/client/status/viewmodel"
)
func legacy(c Client, flag statusFlags) error {
// Set this so we don't overwrite the default terminal colors
// See https://github.com/rivo/tview/blob/master/styles.go
tview.Styles.PrimitiveBackgroundColor = tcell.ColorDefault
tview.Styles.ContrastBackgroundColor = tcell.ColorDefault
tview.Styles.PrimaryTextColor = tcell.ColorDefault
tview.Styles.BorderColor = tcell.ColorDefault
app := tview.NewApplication()
textView := tview.NewTextView()
textView.SetWrap(true)
textView.SetScrollable(true) // so that it allows us to set scroll position
textView.SetScrollBarVisibility(tview.ScrollBarNever)
app.SetRoot(textView, true)
width := (1 << 31) - 1
wrap := false
if isatty.IsTerminal(os.Stdout.Fd()) {
wrap = true
screen, err := tcell.NewScreen()
if err != nil {
return errors.Wrap(err, "get terminal dimensions")
}
if err := screen.Init(); err != nil {
return errors.Wrap(err, "init screen")
}
width, _ = screen.Size()
screen.Fini()
}
paramsMtx := &sync.Mutex{}
params := viewmodel.Params{
Report: nil,
ReportFetchError: nil,
SelectedJob: nil,
FSFilter: func(s string) bool { return true },
DetailViewWidth: width,
DetailViewWrap: wrap,
ShortKeybindingOverview: "",
}
redraw := func() {
textView.Clear()
paramsMtx.Lock()
defer paramsMtx.Unlock()
if params.ReportFetchError != nil {
fmt.Fprintln(textView, params.ReportFetchError.Error())
} else if params.Report != nil {
m := viewmodel.New()
m.Update(params)
for _, j := range m.Jobs() {
if flag.Job != "" && j.Name() != flag.Job {
continue
}
params.SelectedJob = j
m.Update(params)
fmt.Fprintln(textView, m.SelectedJob().FullDescription())
if flag.Job != "" {
break
} else {
hline := strings.Repeat("-", params.DetailViewWidth)
fmt.Fprintln(textView, hline)
}
}
} else {
fmt.Fprintln(textView, "waiting for request results")
}
textView.ScrollToBeginning()
}
app.SetBeforeDrawFunc(func(screen tcell.Screen) bool {
// sync resizes to `params`
paramsMtx.Lock()
_, _, newWidth, _ := textView.GetInnerRect()
if newWidth != params.DetailViewWidth {
params.DetailViewWidth = newWidth
app.QueueUpdateDraw(redraw)
}
paramsMtx.Unlock()
textView.ScrollToBeginning() // has the effect of inhibiting user scrolls
return false
})
go func() {
defer func() {
if err := recover(); err != nil {
app.Suspend(func() {
panic(err)
})
}
}()
for {
st, err := c.Status()
paramsMtx.Lock()
params.Report = st.Jobs
params.ReportFetchError = err
paramsMtx.Unlock()
app.QueueUpdateDraw(redraw)
time.Sleep(flag.Delay)
}
}()
return app.Run()
}

View File

@ -0,0 +1,18 @@
package status
import (
"bytes"
"io"
"os"
)
func raw(c Client) error {
b, err := c.StatusRaw()
if err != nil {
return err
}
if _, err := io.Copy(os.Stdout, bytes.NewReader(b)); err != nil {
return err
}
return nil
}

View File

@ -0,0 +1,18 @@
package viewmodel
import (
"fmt"
)
func ByteCountBinary(b int64) string {
const unit = 1024
if b < unit {
return fmt.Sprintf("%d B", b)
}
div, exp := int64(unit), 0
for n := b / unit; n >= unit; n /= unit {
div *= unit
exp++
}
return fmt.Sprintf("%.1f %ciB", float64(b)/float64(div), "KMGTPE"[exp])
}

View File

@ -0,0 +1,48 @@
package viewmodel
import "time"
type byteProgressMeasurement struct {
time time.Time
val int64
}
type bytesProgressHistory struct {
last *byteProgressMeasurement // pointer as poor man's optional
changeCount int
lastChange time.Time
bpsAvg float64
}
func (p *bytesProgressHistory) Update(currentVal int64) (bytesPerSecondAvg int64, changeCount int) {
if p.last == nil {
p.last = &byteProgressMeasurement{
time: time.Now(),
val: currentVal,
}
return 0, 0
}
if p.last.val != currentVal {
p.changeCount++
p.lastChange = time.Now()
}
if time.Since(p.lastChange) > 3*time.Second {
p.last = nil
return 0, 0
}
deltaV := currentVal - p.last.val
deltaT := time.Since(p.last.time)
rate := float64(deltaV) / deltaT.Seconds()
factor := 0.3
p.bpsAvg = (1-factor)*p.bpsAvg + factor*rate
p.last.time = time.Now()
p.last.val = currentVal
return int64(p.bpsAvg), p.changeCount
}

View File

@ -0,0 +1,576 @@
package viewmodel
import (
"fmt"
"math"
"sort"
"strings"
"time"
"github.com/go-playground/validator/v10"
yaml "github.com/zrepl/yaml-config"
"github.com/zrepl/zrepl/client/status/viewmodel/stringbuilder"
"github.com/zrepl/zrepl/daemon"
"github.com/zrepl/zrepl/daemon/job"
"github.com/zrepl/zrepl/daemon/pruner"
"github.com/zrepl/zrepl/daemon/snapper"
"github.com/zrepl/zrepl/replication/report"
)
type M struct {
jobs map[string]*Job
jobsList []*Job
selectedJob *Job
dateString string
bottomBarStatus string
}
type Job struct {
// long-lived
name string
byteProgress *bytesProgressHistory
lastStatus *job.Status
fulldescription string
}
func New() *M {
return &M{
jobs: make(map[string]*Job),
jobsList: make([]*Job, 0),
selectedJob: nil,
}
}
type FilterFunc func(string) bool
type Params struct {
Report map[string]*job.Status
ReportFetchError error
SelectedJob *Job
FSFilter FilterFunc `validate:"required"`
DetailViewWidth int `validate:"gte=1"`
DetailViewWrap bool
ShortKeybindingOverview string
}
var validate = validator.New()
func (m *M) Update(p Params) {
if err := validate.Struct(p); err != nil {
panic(err)
}
if p.ReportFetchError != nil {
m.bottomBarStatus = fmt.Sprintf("[red::]status fetch: %s", p.ReportFetchError)
} else {
m.bottomBarStatus = p.ShortKeybindingOverview
for jobname, st := range p.Report {
// TODO handle job renames & deletions
j, ok := m.jobs[jobname]
if !ok {
j = &Job{
name: jobname,
byteProgress: &bytesProgressHistory{},
}
m.jobs[jobname] = j
m.jobsList = append(m.jobsList, j)
}
j.lastStatus = st
}
}
// filter out internal jobs
var jobsList []*Job
for _, j := range m.jobsList {
if daemon.IsInternalJobName(j.name) {
continue
}
jobsList = append(jobsList, j)
}
m.jobsList = jobsList
// determinism!
sort.Slice(m.jobsList, func(i, j int) bool {
return strings.Compare(m.jobsList[i].name, m.jobsList[j].name) < 0
})
// try to not lose the selected job
m.selectedJob = nil
for _, j := range m.jobsList {
j.updateFullDescription(p)
if j == p.SelectedJob {
m.selectedJob = j
}
}
m.dateString = time.Now().Format(time.RFC3339)
}
func (m *M) BottomBarStatus() string { return m.bottomBarStatus }
func (m *M) Jobs() []*Job { return m.jobsList }
// may be nil
func (m *M) SelectedJob() *Job { return m.selectedJob }
func (m *M) DateString() string { return m.dateString }
func (j *Job) updateFullDescription(p Params) {
width := p.DetailViewWidth
if !p.DetailViewWrap {
width = 10000000 // FIXME
}
b := stringbuilder.New(stringbuilder.Config{
IndentMultiplier: 3,
Width: width,
})
drawJob(b, j.name, j.lastStatus, j.byteProgress, p.FSFilter)
j.fulldescription = b.String()
}
func (j *Job) JobTreeTitle() string {
return j.name
}
func (j *Job) FullDescription() string {
return j.fulldescription
}
func (j *Job) Name() string {
return j.name
}
func drawJob(t *stringbuilder.B, name string, v *job.Status, history *bytesProgressHistory, fsfilter FilterFunc) {
t.Printf("Job: %s\n", name)
t.Printf("Type: %s\n\n", v.Type)
if v.Type == job.TypePush || v.Type == job.TypePull {
activeStatus, ok := v.JobSpecific.(*job.ActiveSideStatus)
if !ok || activeStatus == nil {
t.Printf("ActiveSideStatus is null")
t.Newline()
return
}
t.Printf("Replication:")
t.AddIndentAndNewline(1)
renderReplicationReport(t, activeStatus.Replication, history, fsfilter)
t.AddIndentAndNewline(-1)
t.Printf("Pruning Sender:")
t.AddIndentAndNewline(1)
renderPrunerReport(t, activeStatus.PruningSender, fsfilter)
t.AddIndentAndNewline(-1)
t.Printf("Pruning Receiver:")
t.AddIndentAndNewline(1)
renderPrunerReport(t, activeStatus.PruningReceiver, fsfilter)
t.AddIndentAndNewline(-1)
if v.Type == job.TypePush {
t.Printf("Snapshotting:")
t.AddIndentAndNewline(1)
renderSnapperReport(t, activeStatus.Snapshotting, fsfilter)
t.AddIndentAndNewline(-1)
}
} else if v.Type == job.TypeSnap {
snapStatus, ok := v.JobSpecific.(*job.SnapJobStatus)
if !ok || snapStatus == nil {
t.Printf("SnapJobStatus is null")
t.Newline()
return
}
t.Printf("Pruning snapshots:")
t.AddIndentAndNewline(1)
renderPrunerReport(t, snapStatus.Pruning, fsfilter)
t.AddIndentAndNewline(-1)
t.Printf("Snapshotting:")
t.AddIndentAndNewline(1)
renderSnapperReport(t, snapStatus.Snapshotting, fsfilter)
t.AddIndentAndNewline(-1)
} else if v.Type == job.TypeSource {
st := v.JobSpecific.(*job.PassiveStatus)
t.Printf("Snapshotting:\n")
t.AddIndent(1)
renderSnapperReport(t, st.Snapper, fsfilter)
t.AddIndentAndNewline(-1)
} else {
t.Printf("No status representation for job type '%s', dumping as YAML", v.Type)
t.Newline()
asYaml, err := yaml.Marshal(v.JobSpecific)
if err != nil {
t.Printf("Error marshaling status to YAML: %s", err)
t.Newline()
return
}
t.Write(string(asYaml))
t.Newline()
}
}
func printFilesystemStatus(t *stringbuilder.B, rep *report.FilesystemReport, active bool, maxFS int) {
expected, replicated, containsInvalidSizeEstimates := rep.BytesSum()
sizeEstimationImpreciseNotice := ""
if containsInvalidSizeEstimates {
sizeEstimationImpreciseNotice = " (some steps lack size estimation)"
}
if rep.CurrentStep < len(rep.Steps) && rep.Steps[rep.CurrentStep].Info.BytesExpected == 0 {
sizeEstimationImpreciseNotice = " (step lacks size estimation)"
}
status := fmt.Sprintf("%s (step %d/%d, %s/%s)%s",
strings.ToUpper(string(rep.State)),
rep.CurrentStep, len(rep.Steps),
ByteCountBinary(replicated), ByteCountBinary(expected),
sizeEstimationImpreciseNotice,
)
activeIndicator := " "
if active {
activeIndicator = "*"
}
t.AddIndent(1)
t.Printf("%s %s %s ",
activeIndicator,
stringbuilder.RightPad(rep.Info.Name, maxFS, " "),
status)
next := ""
if err := rep.Error(); err != nil {
next = err.Err
} else if rep.State != report.FilesystemDone {
if nextStep := rep.NextStep(); nextStep != nil {
if nextStep.IsIncremental() {
next = fmt.Sprintf("next: %s => %s", nextStep.Info.From, nextStep.Info.To)
} else {
next = fmt.Sprintf("next: full send %s", nextStep.Info.To)
}
attribs := []string{}
if nextStep.Info.Resumed {
attribs = append(attribs, "resumed")
}
attribs = append(attribs, fmt.Sprintf("encrypted=%s", nextStep.Info.Encrypted))
next += fmt.Sprintf(" (%s)", strings.Join(attribs, ", "))
} else {
next = "" // individual FSes may still be in planning state
}
}
t.Printf("%s", next)
t.AddIndent(-1)
t.Newline()
}
func renderReplicationReport(t *stringbuilder.B, rep *report.Report, history *bytesProgressHistory, fsfilter FilterFunc) {
if rep == nil {
t.Printf("...\n")
return
}
if rep.WaitReconnectError != nil {
t.PrintfDrawIndentedAndWrappedIfMultiline("Connectivity: %s", rep.WaitReconnectError)
t.Newline()
}
if !rep.WaitReconnectSince.IsZero() {
delta := time.Until(rep.WaitReconnectUntil).Round(time.Second)
if rep.WaitReconnectUntil.IsZero() || delta > 0 {
var until string
if rep.WaitReconnectUntil.IsZero() {
until = "waiting indefinitely"
} else {
until = fmt.Sprintf("hard fail in %s @ %s", delta, rep.WaitReconnectUntil)
}
t.PrintfDrawIndentedAndWrappedIfMultiline("Connectivity: reconnecting with exponential backoff (since %s) (%s)",
rep.WaitReconnectSince, until)
} else {
t.PrintfDrawIndentedAndWrappedIfMultiline("Connectivity: reconnects reached hard-fail timeout @ %s", rep.WaitReconnectUntil)
}
t.Newline()
}
// TODO visualize more than the latest attempt by folding all attempts into one
if len(rep.Attempts) == 0 {
t.Printf("no attempts made yet")
return
} else {
t.Printf("Attempt #%d", len(rep.Attempts))
if len(rep.Attempts) > 1 {
t.Printf(". Previous attempts failed with the following statuses:")
t.AddIndentAndNewline(1)
for i, a := range rep.Attempts[:len(rep.Attempts)-1] {
t.PrintfDrawIndentedAndWrappedIfMultiline("#%d: %s (failed at %s) (ran %s)\n", i+1, a.State, a.FinishAt, a.FinishAt.Sub(a.StartAt))
}
t.AddIndentAndNewline(-1)
} else {
t.Newline()
}
}
latest := rep.Attempts[len(rep.Attempts)-1]
sort.Slice(latest.Filesystems, func(i, j int) bool {
return latest.Filesystems[i].Info.Name < latest.Filesystems[j].Info.Name
})
// apply filter
filtered := make([]*report.FilesystemReport, 0, len(latest.Filesystems))
for _, fs := range latest.Filesystems {
if !fsfilter(fs.Info.Name) {
continue
}
filtered = append(filtered, fs)
}
latest.Filesystems = filtered
t.Printf("Status: %s", latest.State)
t.Newline()
if latest.State == report.AttemptPlanningError {
t.Printf("Problem: ")
t.PrintfDrawIndentedAndWrappedIfMultiline("%s", latest.PlanError)
t.Newline()
} else if latest.State == report.AttemptFanOutError {
t.Printf("Problem: one or more of the filesystems encountered errors")
t.Newline()
}
if latest.State != report.AttemptPlanning && latest.State != report.AttemptPlanningError {
// Draw global progress bar
// Progress: [---------------]
expected, replicated, containsInvalidSizeEstimates := latest.BytesSum()
rate, changeCount := history.Update(replicated)
t.Write("Progress: ")
t.DrawBar(50, replicated, expected, changeCount)
t.Write(fmt.Sprintf(" %s / %s @ %s/s", ByteCountBinary(replicated), ByteCountBinary(expected), ByteCountBinary(rate)))
t.Newline()
if containsInvalidSizeEstimates {
t.Write("NOTE: not all steps could be size-estimated, total estimate is likely imprecise!")
t.Newline()
}
if len(latest.Filesystems) == 0 {
t.Write("NOTE: no filesystems were considered for replication!")
t.Newline()
}
var maxFSLen int
for _, fs := range latest.Filesystems {
if len(fs.Info.Name) > maxFSLen {
maxFSLen = len(fs.Info.Name)
}
}
for _, fs := range latest.Filesystems {
printFilesystemStatus(t, fs, false, maxFSLen) // FIXME bring 'active' flag back
}
}
}
func renderPrunerReport(t *stringbuilder.B, r *pruner.Report, fsfilter FilterFunc) {
if r == nil {
t.Printf("...\n")
return
}
state, err := pruner.StateString(r.State)
if err != nil {
t.Printf("Status: %q (parse error: %q)\n", r.State, err)
return
}
t.Printf("Status: %s", state)
t.Newline()
if r.Error != "" {
t.Printf("Error: %s\n", r.Error)
}
type commonFS struct {
*pruner.FSReport
completed bool
}
all := make([]commonFS, 0, len(r.Pending)+len(r.Completed))
for i := range r.Pending {
all = append(all, commonFS{&r.Pending[i], false})
}
for i := range r.Completed {
all = append(all, commonFS{&r.Completed[i], true})
}
// filter all
filtered := make([]commonFS, 0, len(all))
for _, fs := range all {
if fsfilter(fs.FSReport.Filesystem) {
filtered = append(filtered, fs)
}
}
all = filtered
switch state {
case pruner.Plan:
fallthrough
case pruner.PlanErr:
return
}
if len(all) == 0 {
t.Printf("nothing to do\n")
return
}
var totalDestroyCount, completedDestroyCount int
var maxFSname int
for _, fs := range all {
totalDestroyCount += len(fs.DestroyList)
if fs.completed {
completedDestroyCount += len(fs.DestroyList)
}
if maxFSname < len(fs.Filesystem) {
maxFSname = len(fs.Filesystem)
}
}
// global progress bar
progress := int(math.Round(80 * float64(completedDestroyCount) / float64(totalDestroyCount)))
t.Write("Progress: ")
t.Write("[")
t.Write(stringbuilder.Times("=", progress))
t.Write(">")
t.Write(stringbuilder.Times("-", 80-progress))
t.Write("]")
t.Printf(" %d/%d snapshots", completedDestroyCount, totalDestroyCount)
t.Newline()
sort.SliceStable(all, func(i, j int) bool {
return strings.Compare(all[i].Filesystem, all[j].Filesystem) == -1
})
// Draw a table-like representation of 'all'
for _, fs := range all {
t.Write(stringbuilder.RightPad(fs.Filesystem, maxFSname, " "))
t.Write(" ")
if !fs.SkipReason.NotSkipped() {
t.Printf("skipped: %s\n", fs.SkipReason)
continue
}
if fs.LastError != "" {
if strings.ContainsAny(fs.LastError, "\r\n") {
t.Printf("ERROR:")
t.PrintfDrawIndentedAndWrappedIfMultiline("%s\n", fs.LastError)
} else {
t.PrintfDrawIndentedAndWrappedIfMultiline("ERROR: %s\n", fs.LastError)
}
t.Newline()
continue
}
pruneRuleActionStr := fmt.Sprintf("(destroy %d of %d snapshots)",
len(fs.DestroyList), len(fs.SnapshotList))
if fs.completed {
t.Printf("Completed %s\n", pruneRuleActionStr)
continue
}
t.Write("Pending ") // whitespace is padding 10
if len(fs.DestroyList) == 1 {
t.Write(fs.DestroyList[0].Name)
} else {
t.Write(pruneRuleActionStr)
}
t.Newline()
}
}
func renderSnapperReport(t *stringbuilder.B, r *snapper.Report, fsfilter FilterFunc) {
if r == nil {
t.Printf("<snapshot type does not have a report>\n")
return
}
t.Printf("Status: %s", r.State)
t.Newline()
if r.Error != "" {
t.Printf("Error: %s\n", r.Error)
}
if !r.SleepUntil.IsZero() {
t.Printf("Sleep until: %s\n", r.SleepUntil)
}
sort.Slice(r.Progress, func(i, j int) bool {
return strings.Compare(r.Progress[i].Path, r.Progress[j].Path) == -1
})
dur := func(d time.Duration) string {
return d.Round(100 * time.Millisecond).String()
}
type row struct {
path, state, duration, remainder, hookReport string
}
var widths struct {
path, state, duration int
}
rows := make([]*row, 0, len(r.Progress))
for _, fs := range r.Progress {
if !fsfilter(fs.Path) {
continue
}
r := &row{
path: fs.Path,
state: fs.State.String(),
}
if fs.HooksHadError {
r.hookReport = fs.Hooks // FIXME render here, not in daemon
}
switch fs.State {
case snapper.SnapPending:
r.duration = "..."
r.remainder = ""
case snapper.SnapStarted:
r.duration = dur(time.Since(fs.StartAt))
r.remainder = fmt.Sprintf("snap name: %q", fs.SnapName)
case snapper.SnapDone:
fallthrough
case snapper.SnapError:
r.duration = dur(fs.DoneAt.Sub(fs.StartAt))
r.remainder = fmt.Sprintf("snap name: %q", fs.SnapName)
}
rows = append(rows, r)
if len(r.path) > widths.path {
widths.path = len(r.path)
}
if len(r.state) > widths.state {
widths.state = len(r.state)
}
if len(r.duration) > widths.duration {
widths.duration = len(r.duration)
}
}
for _, r := range rows {
path := stringbuilder.RightPad(r.path, widths.path, " ")
state := stringbuilder.RightPad(r.state, widths.state, " ")
duration := stringbuilder.RightPad(r.duration, widths.duration, " ")
t.Printf("%s %s %s", path, state, duration)
t.PrintfDrawIndentedAndWrappedIfMultiline(" %s", r.remainder)
if r.hookReport != "" {
t.PrintfDrawIndentedAndWrappedIfMultiline("%s", r.hookReport)
}
t.Newline()
}
}

View File

@ -0,0 +1,119 @@
package stringbuilder
import (
"fmt"
"strings"
"github.com/go-playground/validator/v10"
)
type B struct {
// const
indentMultiplier int
// mut
sb *strings.Builder
indent int
width int
x, y int
}
type Config struct {
IndentMultiplier int `validate:"gte=1"`
Width int `validate:"gte=1"`
}
var validate = validator.New()
func New(config Config) *B {
if err := validate.Struct(config); err != nil {
panic(err)
}
return &B{sb: &strings.Builder{}, width: config.Width, indentMultiplier: config.IndentMultiplier}
}
func (b *B) String() string { return b.sb.String() }
func (w *B) Newline() {
w.Write("\n")
}
func (w *B) PrintfDrawIndentedAndWrappedIfMultiline(format string, args ...interface{}) {
whole := fmt.Sprintf(format, args...)
if strings.ContainsAny(whole, "\n\r") {
w.AddIndent(1)
defer w.AddIndent(-1)
}
w.Write(whole)
}
func (w *B) Printf(format string, args ...interface{}) {
whole := fmt.Sprintf(format, args...)
w.Write(whole)
}
func (t *B) AddIndent(delta int) {
t.indent += delta * t.indentMultiplier
}
func (t *B) AddIndentAndNewline(delta int) {
t.indent += delta * t.indentMultiplier
t.Write("\n")
}
func (w *B) Write(s string) {
for _, c := range s {
if c == '\n' {
fmt.Fprint(w.sb, "\n")
w.x = 0
fmt.Fprint(w.sb, Times(" ", w.indent-w.x))
w.x = w.indent
w.y++
continue
}
if w.x >= w.width {
fmt.Fprint(w.sb, "\n")
w.x = 0
fmt.Fprint(w.sb, Times(" ", w.indent-w.x))
w.x = w.indent
}
fmt.Fprintf(w.sb, "%c", c)
w.x++
}
}
func Times(str string, n int) (out string) {
for i := 0; i < n; i++ {
out += str
}
return
}
func RightPad(str string, length int, pad string) string {
if len(str) > length {
return str[:length]
}
return str + strings.Repeat(pad, length-len(str))
}
// changeCount = 0 indicates stall / no progress
func (w *B) DrawBar(length int, bytes, totalBytes int64, changeCount int) {
const arrowPositions = `>\|/`
var completedLength int
if totalBytes > 0 {
completedLength = int(int64(length) * bytes / totalBytes)
if completedLength > length {
completedLength = length
}
} else if totalBytes == bytes {
completedLength = length
}
w.Write("[")
w.Write(Times("=", completedLength))
w.Write(string(arrowPositions[changeCount%len(arrowPositions)]))
w.Write(Times("-", length-completedLength))
w.Write("]")
}

View File

@ -31,6 +31,12 @@ We use the following annotations for classifying changes:
-----
* |break| Change syntax to trigger a job replication, rename ``zrepl signal wakeup JOB`` to ``zrepl signal replication JOB``
* |feature| New ``zrepl status`` UI:
* Interactive job selection.
* Interactively ``zrepl signal`` jobs.
* Filter filesystems in the job view by name.
* An approximation of the old UI is still included as `--mode legacy` but will be removed in a future release of zrepl.
0.3.1
-----

3
go.mod
View File

@ -5,10 +5,12 @@ go 1.12
require (
github.com/fatih/color v1.7.0
github.com/gdamore/tcell v1.2.0
github.com/gdamore/tcell/v2 v2.2.0
github.com/gitchander/permutation v0.0.0-20181107151852-9e56b92e9909
github.com/go-logfmt/logfmt v0.4.0
github.com/go-playground/universal-translator v0.17.0 // indirect
github.com/go-playground/validator v9.31.0+incompatible
github.com/go-playground/validator/v10 v10.4.1
github.com/go-sql-driver/mysql v1.4.1-0.20190907122137-b2c03bcae3d4
github.com/golang/protobuf v1.4.3
github.com/google/uuid v1.1.2
@ -37,6 +39,7 @@ require (
github.com/yudai/gojsondiff v0.0.0-20170107030110-7b1b7adf999d
github.com/yudai/golcs v0.0.0-20170316035057-ecda9a501e82 // go1.12 thinks it needs this
github.com/zrepl/yaml-config v0.0.0-20191220194647-cbb6b0cf4bdd
gitlab.com/tslocum/cview v1.5.3
golang.org/x/net v0.0.0-20210119194325-5f4716e94777
golang.org/x/sync v0.0.0-20190423024810-112230192c58
golang.org/x/sys v0.0.0-20210124154548-22da62e12c0c

30
go.sum
View File

@ -46,6 +46,11 @@ github.com/gdamore/encoding v1.0.0 h1:+7OoQ1Bc6eTm5niUzBa0Ctsh6JbMW6Ra+YNuAtDBdk
github.com/gdamore/encoding v1.0.0/go.mod h1:alR0ol34c49FCSBLjhosxzcPHQbf2trDkoo5dl+VrEg=
github.com/gdamore/tcell v1.2.0 h1:ikixzsxc8K8o3V2/CEmyoEW8mJZaNYQQ3NP3VIQdUe4=
github.com/gdamore/tcell v1.2.0/go.mod h1:Hjvr+Ofd+gLglo7RYKxxnzCBmev3BzsS67MebKS4zMM=
github.com/gdamore/tcell v1.4.0 h1:vUnHwJRvcPQa3tzi+0QI4U9JINXYJlOz9yiaiPQ2wMU=
github.com/gdamore/tcell/v2 v2.0.0-dev/go.mod h1:vSVL/GV5mCSlPC6thFP5kfOFdM9MGZcalipmpTxTgQA=
github.com/gdamore/tcell/v2 v2.1.1-0.20201225194624-29bb185874fd/go.mod h1:vSVL/GV5mCSlPC6thFP5kfOFdM9MGZcalipmpTxTgQA=
github.com/gdamore/tcell/v2 v2.2.0 h1:vSyEgKwraXPSOkvCk7IwOSyX+Pv3V2cV9CikJMXg4U4=
github.com/gdamore/tcell/v2 v2.2.0/go.mod h1:cTTuF84Dlj/RqmaCIV5p4w8uG1zWdk0SF6oBpwHp4fU=
github.com/gitchander/permutation v0.0.0-20181107151852-9e56b92e9909 h1:9NC8seTx6/zRmMTAdsHj/uOMi0EGHGQtjyLafBjk77Q=
github.com/gitchander/permutation v0.0.0-20181107151852-9e56b92e9909/go.mod h1:lP+DW8LR6Rw3ru9Vo2/y/3iiLaLWmofYql/va+7zJOk=
github.com/go-critic/go-critic v0.3.4/go.mod h1:AHR42Lk/E/aOznsrYdMYeIQS5RH10HZHSqP+rD6AJrc=
@ -59,12 +64,15 @@ github.com/go-logfmt/logfmt v0.3.0/go.mod h1:Qt1PoO58o5twSAckw1HlFXLmHsOX5/0LbT9
github.com/go-logfmt/logfmt v0.4.0 h1:MP4Eh7ZCb31lleYCFuwm0oe4/YGak+5l1vA2NOE80nA=
github.com/go-logfmt/logfmt v0.4.0/go.mod h1:3RMwSq7FuexP4Kalkev3ejPJsZTpXXBr9+V4qmtdjCk=
github.com/go-ole/go-ole v1.2.1/go.mod h1:7FAglXiTm7HKlQRDeOQ6ZNUHidzCWXuZWq/1dTyBNF8=
github.com/go-playground/assert/v2 v2.0.1/go.mod h1:VDjEfimB/XKnb+ZQfWdccd7VUvScMdVu0Titje2rxJ4=
github.com/go-playground/locales v0.13.0 h1:HyWk6mgj5qFqCT5fjGBuRArbVDfE4hi8+e8ceBS/t7Q=
github.com/go-playground/locales v0.13.0/go.mod h1:taPMhCMXrRLJO55olJkUXHZBHCxTMfnGwq/HNwmWNS8=
github.com/go-playground/universal-translator v0.17.0 h1:icxd5fm+REJzpZx7ZfpaD876Lmtgy7VtROAbHHXk8no=
github.com/go-playground/universal-translator v0.17.0/go.mod h1:UkSxE5sNxxRwHyU+Scu5vgOQjsIJAF8j9muTVoKLVtA=
github.com/go-playground/validator v9.31.0+incompatible h1:UA72EPEogEnq76ehGdEDp4Mit+3FDh548oRqwVgNsHA=
github.com/go-playground/validator v9.31.0+incompatible/go.mod h1:yrEkQXlcI+PugkyDjY2bRrL/UBU4f3rvrgkN3V8JEig=
github.com/go-playground/validator/v10 v10.4.1 h1:pH2c5ADXtd66mxoE0Zm9SUhxE20r7aM3F26W0hOn+GE=
github.com/go-playground/validator/v10 v10.4.1/go.mod h1:nlOn6nFhuKACm19sB/8EGNn9GlaMV7XkbRSipzJ0Ii4=
github.com/go-sql-driver/mysql v1.4.1-0.20190907122137-b2c03bcae3d4 h1:0suja/iKSDbEIYLbrS/8C7iArJiWpgCNcR+zwAHu7Ig=
github.com/go-sql-driver/mysql v1.4.1-0.20190907122137-b2c03bcae3d4/go.mod h1:zAC/RDZ24gD3HViQzih4MyKcchzm+sOG5ZlKdlhCg5w=
github.com/go-stack/stack v1.8.0/go.mod h1:v0f6uXyyMGvRgIKkXu+yp6POWl0qKG85gN/melR3HDY=
@ -177,6 +185,7 @@ github.com/kr/pretty v0.1.0/go.mod h1:dAy3ld7l9f0ibDNOQOHHMYYIIbhfbHSm3C4ZsoJORN
github.com/kr/pty v1.1.1/go.mod h1:pFQYn66WHrOpPYNljwOMqo10TkYh1fy3cYio2l3bCsQ=
github.com/kr/text v0.1.0 h1:45sCR5RtlFHMR4UwH9sdQ5TC8v0qDQCHnXt+kaKSTVE=
github.com/kr/text v0.1.0/go.mod h1:4Jbv+DJW3UT/LiOwJeYQe1efqtUx/iVham/4vfdArNI=
github.com/leodido/go-urn v1.2.0/go.mod h1:+8+nEpDfqqsY+g338gtMEUOtuK+4dEMhiQEgxpxOKII=
github.com/leodido/go-urn v1.2.1 h1:BqpAaACuzVSgi/VLzGZIobT2z4v53pjosyNd9Yv6n/w=
github.com/leodido/go-urn v1.2.1/go.mod h1:zt4jvISO2HfUBqxjfIshjdMTYS56ZS/qv49ictyFfxY=
github.com/lib/pq v1.2.0 h1:LXpIM/LZ5xGFhOpXAQUIMM1HdyqzVYM13zNdjCEEcA0=
@ -184,6 +193,8 @@ github.com/lib/pq v1.2.0/go.mod h1:5WUZQaWbwv1U+lTReE5YruASi9Al49XbQIvNi/34Woo=
github.com/logrusorgru/aurora v0.0.0-20181002194514-a7b3b318ed4e/go.mod h1:7rIyQOR62GCctdiQpZ/zOJlFyk6y+94wXzv6RNZgaR4=
github.com/lucasb-eyer/go-colorful v1.0.2 h1:mCMFu6PgSozg9tDNMMK3g18oJBX7oYGrC09mS6CXfO4=
github.com/lucasb-eyer/go-colorful v1.0.2/go.mod h1:0MS4r+7BZKSJ5mw4/S5MPN+qHFF1fYclkSPilDOKW0s=
github.com/lucasb-eyer/go-colorful v1.0.3 h1:QIbQXiugsb+q10B+MI+7DI1oQLdmnep86tWFlaaUAac=
github.com/lucasb-eyer/go-colorful v1.0.3/go.mod h1:R4dSotOR9KMtayYi1e77YzuveK+i7ruzyGqttikkLy0=
github.com/magiconair/properties v1.7.6/go.mod h1:PppfXfuXeibc/6YijjN8zIbojt8czPbwD3XqdrwzmxQ=
github.com/magiconair/properties v1.8.0/go.mod h1:PppfXfuXeibc/6YijjN8zIbojt8czPbwD3XqdrwzmxQ=
github.com/mattn/go-colorable v0.0.9 h1:UVL0vNpWh04HeJXV0KLcaT7r06gOH2l4OW6ddYRUIY4=
@ -196,6 +207,10 @@ github.com/mattn/go-isatty v0.0.8 h1:HLtExJ+uU2HOZ+wI0Tt5DtUDrx8yhUqDcp7fYERX4CE
github.com/mattn/go-isatty v0.0.8/go.mod h1:Iq45c/XA43vh69/j3iqttzPXn0bhXyGjM0Hdxcsrc5s=
github.com/mattn/go-runewidth v0.0.4 h1:2BvfKmzob6Bmd4YsL0zygOqfdFnK7GR4QL06Do4/p7Y=
github.com/mattn/go-runewidth v0.0.4/go.mod h1:LwmH8dsx7+W8Uxz3IHJYH5QSwggIsqBzpuz5H//U1FU=
github.com/mattn/go-runewidth v0.0.7/go.mod h1:H031xJmbD/WCDINGzjvQ9THkh0rPKHF+m2gUSrubnMI=
github.com/mattn/go-runewidth v0.0.9/go.mod h1:H031xJmbD/WCDINGzjvQ9THkh0rPKHF+m2gUSrubnMI=
github.com/mattn/go-runewidth v0.0.10 h1:CoZ3S2P7pvtP45xOtBw+/mDL2z0RKI576gSkzRRpdGg=
github.com/mattn/go-runewidth v0.0.10/go.mod h1:RAqKPSqVFrSLVXbA8x7dzmKdmGzieGRCM46jaSJTDAk=
github.com/mattn/goveralls v0.0.2/go.mod h1:8d1ZMHsd7fW6IRPKQh46F2WRpyib5/X4FOpevwGNQEw=
github.com/matttproud/golang_protobuf_extensions v1.0.0/go.mod h1:D8He9yQNgCq6Z5Ld7szi9bcBfOoFv/3dc6xSMkL2PC0=
github.com/matttproud/golang_protobuf_extensions v1.0.1 h1:4hp9jkHxhMHkqkrB3Ix0jegS5sx/RkqARlsWZ6pIwiU=
@ -264,6 +279,9 @@ github.com/prometheus/procfs v0.0.2/go.mod h1:TjEm7ze935MbeOT/UhFTIMYKhuLP4wbCsT
github.com/prometheus/procfs v0.0.5 h1:3+auTFlqw+ZaQYJARz6ArODtkaIwtvBTx3N2NehQlL8=
github.com/prometheus/procfs v0.0.5/go.mod h1:4A/X28fw3Fc593LaREMrKMqOKvUAntwMDaekg4FpcdQ=
github.com/quasilyte/go-consistent v0.0.0-20190521200055-c6f3937de18c/go.mod h1:5STLWrekHfjyYwxBRVRXNOSewLJ3PWfDJd1VyTS21fI=
github.com/rivo/uniseg v0.1.0/go.mod h1:J6wj4VEh+S6ZtnVlnTBMWIodfgj8LQOQFoIToxlJtxc=
github.com/rivo/uniseg v0.2.0 h1:S1pD9weZBuJdFmowNwbpi7BJ8TNftyUImj/0WQi72jY=
github.com/rivo/uniseg v0.2.0/go.mod h1:J6wj4VEh+S6ZtnVlnTBMWIodfgj8LQOQFoIToxlJtxc=
github.com/rogpeppe/go-internal v1.1.0/go.mod h1:M8bDsm7K2OlrFYOpmOWEs/qY81heoFRclV5y23lUDJ4=
github.com/rogpeppe/go-internal v1.2.1/go.mod h1:M8bDsm7K2OlrFYOpmOWEs/qY81heoFRclV5y23lUDJ4=
github.com/ryanuber/go-glob v0.0.0-20170128012129-256dc444b735/go.mod h1:807d1WSdnB0XRJzKNil9Om6lcp/3a0v4qIHxIXzX/Yc=
@ -332,11 +350,17 @@ github.com/zrepl/yaml-config v0.0.0-20190928121844-af7ca3f8448f/go.mod h1:JmNwis
github.com/zrepl/yaml-config v0.0.0-20191220194647-cbb6b0cf4bdd h1:SSo67WLS+99QESvbW8Meibz7zCrxshP71U9dH5KOCXM=
github.com/zrepl/yaml-config v0.0.0-20191220194647-cbb6b0cf4bdd/go.mod h1:JmNwisZzOvW4GfpfLvhZ+gtyKLsIiA+WC+wNKJGJaFg=
github.com/zrepl/zrepl v0.2.0/go.mod h1:M3Zv2IGSO8iYpUjsZD6ayZ2LHy7zyMfzet9XatKOrZ8=
gitlab.com/tslocum/cbind v0.1.4 h1:cbZXPPcieXspk8cShoT6efz7HAT8yMNQcofYWNizis4=
gitlab.com/tslocum/cbind v0.1.4/go.mod h1:RvwYE3auSjBNlCmWeGspzn+jdLUVQ8C2QGC+0nP9ChI=
gitlab.com/tslocum/cview v1.5.3 h1:6OTCtIUp1EkfGeLqQFRHtW8ynMJ66BhoBwuW8oZ84AQ=
gitlab.com/tslocum/cview v1.5.3/go.mod h1:k/eLWRIF3B26VLDgtRRPkjLUXmcCsy+YCSPEAtNQgIY=
golang.org/x/crypto v0.0.0-20180904163835-0709b304e793/go.mod h1:6SG95UA2DQfeDnfUPMdvaQW0Q7yPrPDi9nlGo2tz2b4=
golang.org/x/crypto v0.0.0-20181203042331-505ab145d0a9/go.mod h1:6SG95UA2DQfeDnfUPMdvaQW0Q7yPrPDi9nlGo2tz2b4=
golang.org/x/crypto v0.0.0-20190308221718-c2843e01d9a2/go.mod h1:djNgcEr1/C05ACkg1iLfiJU5Ep61QUkGW8qpdssI0+w=
golang.org/x/crypto v0.0.0-20190313024323-a1f597ede03a/go.mod h1:djNgcEr1/C05ACkg1iLfiJU5Ep61QUkGW8qpdssI0+w=
golang.org/x/crypto v0.0.0-20190325154230-a5d413f7728c/go.mod h1:djNgcEr1/C05ACkg1iLfiJU5Ep61QUkGW8qpdssI0+w=
golang.org/x/crypto v0.0.0-20200622213623-75b288015ac9 h1:psW17arqaxU48Z5kZ0CQnkZWQJsqcURM6tKiBApRjXI=
golang.org/x/crypto v0.0.0-20200622213623-75b288015ac9/go.mod h1:LzIPMQfyMNhhGPhUkYOs5KpL4U8rLKemX1yGLhDgUto=
golang.org/x/exp v0.0.0-20180321215751-8460e604b9de/go.mod h1:CJ0aWSM057203Lf6IL+f9T1iT9GByDxfZKAQTCR3kQA=
golang.org/x/exp v0.0.0-20180807140117-3d87b88a115f/go.mod h1:CJ0aWSM057203Lf6IL+f9T1iT9GByDxfZKAQTCR3kQA=
golang.org/x/exp v0.0.0-20190121172915-509febef88a4/go.mod h1:CJ0aWSM057203Lf6IL+f9T1iT9GByDxfZKAQTCR3kQA=
@ -357,6 +381,7 @@ golang.org/x/net v0.0.0-20181114220301-adae6a3d119a/go.mod h1:mL1N/T3taQHkDXs73r
golang.org/x/net v0.0.0-20190213061140-3a22650c66bd/go.mod h1:mL1N/T3taQHkDXs73rZJwtUhF3w3ftmwwsq0BUmARs4=
golang.org/x/net v0.0.0-20190311183353-d8887717615a/go.mod h1:t9HGtf8HONx5eT2rtn7q6eTqICYqUVnKs3thJo3Qplg=
golang.org/x/net v0.0.0-20190313220215-9f648a60d977/go.mod h1:t9HGtf8HONx5eT2rtn7q6eTqICYqUVnKs3thJo3Qplg=
golang.org/x/net v0.0.0-20190404232315-eb5bcb51f2a3/go.mod h1:t9HGtf8HONx5eT2rtn7q6eTqICYqUVnKs3thJo3Qplg=
golang.org/x/net v0.0.0-20190613194153-d28f0bde5980 h1:dfGZHvZk057jK2MCeWus/TowKpJ8y4AmooUzdBSR9GU=
golang.org/x/net v0.0.0-20190613194153-d28f0bde5980/go.mod h1:z5CRVTTTmAJ677TzLLGU+0bjPO0LkuOLi4/5GtJWs/s=
golang.org/x/net v0.0.0-20210119194325-5f4716e94777 h1:003p0dJM77cxMSyCPFphvZf/Y5/NXf5fzg6ufd1/Oew=
@ -376,6 +401,7 @@ golang.org/x/sys v0.0.0-20181205085412-a5c9d58dba9a/go.mod h1:STP8DvDyc/dI5b8T5h
golang.org/x/sys v0.0.0-20190215142949-d0b11bdaac8a/go.mod h1:STP8DvDyc/dI5b8T5hshtkjS+E42TnysNCUPdjciGhY=
golang.org/x/sys v0.0.0-20190222072716-a9d3bda3a223/go.mod h1:STP8DvDyc/dI5b8T5hshtkjS+E42TnysNCUPdjciGhY=
golang.org/x/sys v0.0.0-20190312061237-fead79001313/go.mod h1:h1NjWce9XRLGQEsW7wpKNCjG9DtNlClVuFLEZdDNbEs=
golang.org/x/sys v0.0.0-20190412213103-97732733099d/go.mod h1:h1NjWce9XRLGQEsW7wpKNCjG9DtNlClVuFLEZdDNbEs=
golang.org/x/sys v0.0.0-20190422165155-953cdadca894/go.mod h1:h1NjWce9XRLGQEsW7wpKNCjG9DtNlClVuFLEZdDNbEs=
golang.org/x/sys v0.0.0-20190626150813-e07cf5db2756 h1:9nuHUbU8dRnRRfj9KjWUVrJeoexdbeMjttk6Oh1rD10=
golang.org/x/sys v0.0.0-20190626150813-e07cf5db2756/go.mod h1:h1NjWce9XRLGQEsW7wpKNCjG9DtNlClVuFLEZdDNbEs=
@ -383,10 +409,14 @@ golang.org/x/sys v0.0.0-20191010194322-b09406accb47 h1:/XfQ9z7ib8eEJX2hdgFTZJ/nt
golang.org/x/sys v0.0.0-20191010194322-b09406accb47/go.mod h1:h1NjWce9XRLGQEsW7wpKNCjG9DtNlClVuFLEZdDNbEs=
golang.org/x/sys v0.0.0-20191026070338-33540a1f6037 h1:YyJpGZS1sBuBCzLAR1VEpK193GlqGZbnPFnPV/5Rsb4=
golang.org/x/sys v0.0.0-20191026070338-33540a1f6037/go.mod h1:h1NjWce9XRLGQEsW7wpKNCjG9DtNlClVuFLEZdDNbEs=
golang.org/x/sys v0.0.0-20201013132646-2da7054afaeb/go.mod h1:h1NjWce9XRLGQEsW7wpKNCjG9DtNlClVuFLEZdDNbEs=
golang.org/x/sys v0.0.0-20201119102817-f84b799fce68/go.mod h1:h1NjWce9XRLGQEsW7wpKNCjG9DtNlClVuFLEZdDNbEs=
golang.org/x/sys v0.0.0-20210113181707-4bcb84eeeb78/go.mod h1:h1NjWce9XRLGQEsW7wpKNCjG9DtNlClVuFLEZdDNbEs=
golang.org/x/sys v0.0.0-20210124154548-22da62e12c0c h1:VwygUrnw9jn88c4u8GD3rZQbqrP/tgas88tPUbBxQrk=
golang.org/x/sys v0.0.0-20210124154548-22da62e12c0c/go.mod h1:h1NjWce9XRLGQEsW7wpKNCjG9DtNlClVuFLEZdDNbEs=
golang.org/x/term v0.0.0-20201126162022-7de9c90e9dd1/go.mod h1:bj7SfCRtBDWHUb9snDiAeCFNEtKQo2Wmx5Cou7ajbmo=
golang.org/x/term v0.0.0-20201210144234-2321bbc49cbf h1:MZ2shdL+ZM/XzY3ZGOnh4Nlpnxz5GSOhOmtHo3iPU6M=
golang.org/x/term v0.0.0-20201210144234-2321bbc49cbf/go.mod h1:bj7SfCRtBDWHUb9snDiAeCFNEtKQo2Wmx5Cou7ajbmo=
golang.org/x/text v0.0.0-20170915090833-1cbadb444a80/go.mod h1:NqM8EUOU14njkJ3fqMW+pc6Ldnwhi/IjpwHt7yyuwOQ=
golang.org/x/text v0.3.0 h1:g61tztE5qeGQ89tm6NTjjM9VPIm088od1l6aSorWRWg=
golang.org/x/text v0.3.0/go.mod h1:NqM8EUOU14njkJ3fqMW+pc6Ldnwhi/IjpwHt7yyuwOQ=

View File

@ -4,12 +4,13 @@ package main
import (
"github.com/zrepl/zrepl/cli"
"github.com/zrepl/zrepl/client"
"github.com/zrepl/zrepl/client/status"
"github.com/zrepl/zrepl/daemon"
)
func init() {
cli.AddSubcommand(daemon.DaemonCmd)
cli.AddSubcommand(client.StatusCmd)
cli.AddSubcommand(status.Subcommand)
cli.AddSubcommand(client.SignalCmd)
cli.AddSubcommand(client.StdinserverCmd)
cli.AddSubcommand(client.ConfigcheckCmd)

98
util/choices/choices.go Normal file
View File

@ -0,0 +1,98 @@
// Package choice implements a flag.Value type that accepts a set of choices.
//
// See test cases or grep the code base for usage hints.
package choices
import (
"flag"
"fmt"
"strings"
)
type Choices struct {
choices map[string]interface{}
typeString string
value interface{}
}
var _ flag.Value = (*Choices)(nil)
func new(pairs ...interface{}) Choices {
if (len(pairs) % 2) != 0 {
panic("must provide a sequence of key value pairs")
}
c := Choices{
choices: make(map[string]interface{}, len(pairs)/2),
value: nil,
}
for i := 0; i < len(pairs); {
key, ok := pairs[i].(string)
if !ok {
panic(fmt.Sprintf("argument %d is %T but should be a string, value: %#v", i, pairs[i], pairs[i]))
}
c.choices[key] = pairs[i+1]
i += 2
}
c.typeString = strings.Join(c.choicesList(true), ",") // overrideable by setter
return c
}
func (c *Choices) Init(pairs ...interface{}) {
*c = new(pairs...)
}
func (c Choices) choicesList(escaped bool) []string {
keys := make([]string, len(c.choices))
i := 0
for k := range c.choices {
e := k
if escaped {
e = fmt.Sprintf("%q", k)
}
keys[i] = e
i += 1
}
return keys
}
func (c Choices) Usage() string {
return fmt.Sprintf("one of %s", strings.Join(c.choicesList(true), ","))
}
func (c Choices) InputForChoice(v interface{}) (string, error) {
for input, choice := range c.choices {
if choice == v {
return input, nil
}
}
return "", fmt.Errorf("choice not registered at .Init(): %v", v)
}
func (c *Choices) SetDefaultValue(v interface{}) {
c.value = v
}
func (c Choices) Value() interface{} {
return c.value
}
func (c *Choices) Set(input string) error {
v, ok := c.choices[input]
if !ok {
return fmt.Errorf("invalid value %q: must be one of %s", input, c.Usage())
}
c.value = v
return nil
}
func (c *Choices) String() string {
return "" // c.value.(fmt.Stringer).String()
}
func (c *Choices) SetTypeString(ts string) {
c.typeString = ts
}
func (c *Choices) Type() string {
return c.typeString
}

View File

@ -0,0 +1,50 @@
package choices_test
import (
"bytes"
"flag"
"os"
"testing"
"github.com/stretchr/testify/require"
"github.com/zrepl/zrepl/util/choices"
)
func TestChoices(t *testing.T) {
var c choices.Choices
fs := flag.NewFlagSet("testset", flag.ContinueOnError)
c.Init("append", os.O_APPEND, "overwrite", os.O_TRUNC|os.O_CREATE)
fs.Var(&c, "mode", c.Usage())
var o bytes.Buffer
fs.SetOutput(&o)
fs.Usage()
usage := o.String()
o.Reset()
t.Logf("usage:\n%s", usage)
require.Contains(t, usage, "\"append\"")
require.Contains(t, usage, "\"overwrite\"")
err := fs.Parse([]string{"-mode", "append"})
require.NoError(t, err)
o.Reset()
require.Equal(t, os.O_APPEND, c.Value())
c.SetDefaultValue(nil)
err = fs.Parse([]string{})
require.NoError(t, err)
o.Reset()
require.Nil(t, c.Value())
// a little whitebox testing: this is allowed ATM, we don't check that the default value was specified as a choice in init
c.SetDefaultValue(os.O_RDWR)
err = fs.Parse([]string{})
require.NoError(t, err)
o.Reset()
require.Equal(t, os.O_RDWR, c.Value())
}