logtree: implement
This implements logtree, as per go/logtree .
Some API changes are present, the design doc will be updated to reflect
these and the rationale behind the changes.
This implementation is missing 'raw' log functionality, this will be
added in a diff on top (as the implementation is trivial, but we want
to keep this diff as simple as possible).
Test Plan: covered by tests
X-Origin-Diff: phab/D624
GitOrigin-RevId: 6d1e0fb16f47e4b0dc9a18765cecb9314bbcb441
diff --git a/core/pkg/logtree/BUILD.bazel b/core/pkg/logtree/BUILD.bazel
index c3d1bd8..ed9fda4 100644
--- a/core/pkg/logtree/BUILD.bazel
+++ b/core/pkg/logtree/BUILD.bazel
@@ -1,8 +1,28 @@
-load("@io_bazel_rules_go//go:def.bzl", "go_library")
+load("@io_bazel_rules_go//go:def.bzl", "go_library", "go_test")
go_library(
name = "go_default_library",
- srcs = ["leveled.go"],
+ srcs = [
+ "doc.go",
+ "journal.go",
+ "journal_entry.go",
+ "journal_subscriber.go",
+ "leveled.go",
+ "logtree.go",
+ "logtree_access.go",
+ "logtree_publisher.go",
+ "payload.go",
+ ],
importpath = "git.monogon.dev/source/nexantic.git/core/pkg/logtree",
visibility = ["//visibility:public"],
)
+
+go_test(
+ name = "go_default_test",
+ srcs = [
+ "journal_test.go",
+ "logtree_test.go",
+ "payload_test.go",
+ ],
+ embed = [":go_default_library"],
+)
diff --git a/core/pkg/logtree/doc.go b/core/pkg/logtree/doc.go
new file mode 100644
index 0000000..5445e2e
--- /dev/null
+++ b/core/pkg/logtree/doc.go
@@ -0,0 +1,98 @@
+// Copyright 2020 The Monogon Project Authors.
+//
+// SPDX-License-Identifier: Apache-2.0
+//
+// Licensed under the Apache License, Version 2.0 (the "License");
+// you may not use this file except in compliance with the License.
+// You may obtain a copy of the License at
+//
+// http://www.apache.org/licenses/LICENSE-2.0
+//
+// Unless required by applicable law or agreed to in writing, software
+// distributed under the License is distributed on an "AS IS" BASIS,
+// WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+// See the License for the specific language governing permissions and
+// limitations under the License.
+
+/*
+Package logtree implements a tree-shaped logger for debug events. It provides log publishers (ie. Go code) with a
+glog-like API, with loggers placed in a hierarchical structure defined by a dot-delimited path (called a DN, short for
+Distinguished Name).
+
+ tree.MustLeveledFor("foo.bar.baz").Warningf("Houston, we have a problem: %v", err)
+
+Logs in this context are unstructured, operational and developer-centric human readable text messages presented as lines
+of text to consumers, with some attached metadata. Logtree does not deal with 'structured' logs as some parts of the
+industry do, and instead defers any machine-readable logs to either be handled by metrics systems like Prometheus or
+event sourcing systems like Kafka.
+
+Tree Structure
+
+As an example, consider application that produces logs with the following DNs:
+
+ listener.http
+ listener.grpc
+ svc
+ svc.cache
+ svc.cache.gc
+
+This would correspond to a tree as follows:
+
+ .------.
+ | "" |
+ | (root) |
+ '------'
+ .----------------' '------.
+ .--------------. .---------------.
+ | svc | | listener |
+ '--------------' '---------------'
+ | .----' '----.
+ .--------------. .---------------. .---------------.
+ | svc.cache | | listener.http | | listener.grpc |
+ '--------------' '---------------' '---------------'
+ |
+ .--------------.
+ | svc.cache.gc |
+ '--------------'
+
+In this setup, every DN acts as a separate logging target, each with its own retention policy and quota. Logging to a DN
+under foo.bar does NOT automatically log to foo - all tree mechanisms are applied on log access by consumers. Loggers
+are automatically created on first use, and importantly, can be created at any time, and will automatically be created
+if a sub-DN is created that requires a parent DN to exist first. Note, for instance, that a `listener` logging node was
+created even though the example application only logged to `listener.http` and `listener.grpc`.
+
+An implicit root node is always present in the tree, accessed by DN "" (an empty string). All other logger nodes are
+children (or transitive children) of the root node.
+
+Log consumers (application code that reads the log and passes them on to operators, or ships them off for aggregation in
+other systems) to select subtrees of logs for readout. In the example tree, a consumer could select to either read all
+logs of the entire tree, just a single DN (like svc), or a subtree (like everything under listener, ie. messages emitted
+to listener.http and listener.grpc).
+
+Log Producer API
+
+As part of the glog-like logging API available to producers, the following metadata is attached to emitted logs in
+addition to the DN of the logger to which the log entry was emitted:
+
+ - timestamp at which the entry was emitted
+ - a severity level (one of FATAL, ERROR, WARN or INFO)
+ - a source of the message (file name and line number)
+
+In addition, the logger mechanism supports a variable verbosity level (so-called 'V-logging') that can be set at every
+node of the tree. For more information about the producer-facing logging API, see the documentation of the LeveledLogger
+interface, which is the main interface exposed to log producers.
+
+Log Access API
+
+The Log Access API is mostly exposed via a single function on the LogTree struct: Read. It allows access to log entries
+that have been already buffered inside LogTree and to subscribe to receive future entries over a channel. As outlined
+earlier, any access can specify whether it is just interested in a single logger (addressed by DN), or a subtree of
+loggers.
+
+Due to the current implementation of the logtree, subtree accesses of backlogged data is significantly slower than
+accessing data of just one DN, or the whole tree (as every subtree backlog access performs a scan on all logged data).
+Thus, log consumers should be aware that it is much better to stream and buffer logs specific to some long-standing
+logging request on their own, rather than repeatedly perform reads of a subtree backlog.
+
+*/
+package logtree
diff --git a/core/pkg/logtree/journal.go b/core/pkg/logtree/journal.go
new file mode 100644
index 0000000..d65a2f4
--- /dev/null
+++ b/core/pkg/logtree/journal.go
@@ -0,0 +1,207 @@
+// Copyright 2020 The Monogon Project Authors.
+//
+// SPDX-License-Identifier: Apache-2.0
+//
+// Licensed under the Apache License, Version 2.0 (the "License");
+// you may not use this file except in compliance with the License.
+// You may obtain a copy of the License at
+//
+// http://www.apache.org/licenses/LICENSE-2.0
+//
+// Unless required by applicable law or agreed to in writing, software
+// distributed under the License is distributed on an "AS IS" BASIS,
+// WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+// See the License for the specific language governing permissions and
+// limitations under the License.
+
+package logtree
+
+import (
+ "fmt"
+ "strings"
+ "sync"
+)
+
+// DN is the Distinguished Name, a dot-delimited path used to address loggers within a LogTree. For example, "foo.bar"
+// designates the 'bar' logger node under the 'foo' logger node under the root node of the logger. An empty string is
+// the root node of the tree.
+type DN string
+
+// Parth return the parts of a DN, ie. all the elements of the dot-delimited DN path. For the root node, an empty list
+// will be returned. An error will be returned if the DN is invalid (contains empty parts, eg. `foo..bar`, `.foo` or
+// `foo.`.
+func (d DN) Path() ([]string, error) {
+ if d == "" {
+ return nil, nil
+ }
+ parts := strings.Split(string(d), ".")
+ for _, p := range parts {
+ if p == "" {
+ return nil, fmt.Errorf("invalid DN")
+ }
+ }
+ return parts, nil
+}
+
+// journal is the main log recording structure of logtree. It manages linked lists containing the actual log entries,
+// and implements scans across them. It does not understand the hierarchical nature of logtree, and instead sees all
+// entries as part of a global linked list and a local linked list for a given DN.
+//
+// The global linked list is represented by the head/tail pointers in journal and nextGlobal/prevGlobal pointers in
+// entries. The local linked lists are represented by heads[DN]/tails[DN] pointers in journal and nextLocal/prevLocal
+// pointers in entries:
+//
+// .------------. .------------. .------------.
+// | dn: A.B | | dn: Z | | dn: A.B |
+// | time: 1 | | time: 2 | | time: 3 |
+// |------------| |------------| |------------|
+// | nextGlobal :------->| nextGlobal :------->| nextGlobal :--> nil
+// nil <-: prevGlobal |<-------: prevGlobal |<-------| prevGlobal |
+// |------------| |------------| n |------------|
+// | nextLocal :---. n | nextLocal :->i .-->| nextLocal :--> nil
+// nil <-: prevLocal |<--: i<-: prevLocal | l :---| prevLocal |
+// '------------' | l '------------' | '------------'
+// ^ '----------------------' ^
+// | ^ |
+// | | |
+// ( head ) ( tails[Z] ) ( tail )
+// ( heads[A.B] ) ( heads[Z] ) ( tails[A.B] )
+//
+type journal struct {
+ // mu locks the rest of the structure. It must be taken during any operation on the journal.
+ mu sync.RWMutex
+
+ // tail is the side of the global linked list that contains the newest log entry, ie. the one that has been pushed
+ // the most recently. It can be nil when no log entry has yet been pushed. The global linked list contains all log
+ // entries pushed to the journal.
+ tail *entry
+ // head is the side of the global linked list that contains the oldest log entry. It can be nil when no log entry
+ // has yet been pushed.
+ head *entry
+
+ // tails are the tail sides of a local linked list for a given DN, ie. the sides that contain the newest entry. They
+ // are nil if there are no log entries for that DN.
+ tails map[DN]*entry
+ // heads are the head sides of a local linked list for a given DN, ie. the sides that contain the oldest entry. They
+ // are nil if there are no log entries for that DN.
+ heads map[DN]*entry
+
+ // quota is a map from DN to quota structure, representing the quota policy of a particular DN-designated logger.
+ quota map[DN]*quota
+
+ // subscribers are observer to logs. New log entries get emitted to channels present in the subscriber structure,
+ // after filtering them through subscriber-provided filters (eg. to limit events to subtrees that interest that
+ // particular subscriber).
+ subscribers []*subscriber
+}
+
+// newJournal creates a new empty journal. All journals are independent from eachother, and as such, all LogTrees are
+// also independent.
+func newJournal() *journal {
+ return &journal{
+ tails: make(map[DN]*entry),
+ heads: make(map[DN]*entry),
+
+ quota: make(map[DN]*quota),
+ }
+}
+
+// filter is a predicate that returns true if a log subscriber or reader is interested in a log entry at a given
+// severity and logged to a given DN.
+type filter func(origin DN, severity Severity) bool
+
+// filterALl returns a filter that accepts all log entries.
+func filterAll() filter {
+ return func(origin DN, _ Severity) bool { return true }
+}
+
+// filterExact returns a filter that accepts only log entries at a given exact DN. This filter should not be used in
+// conjunction with journal.scanEntries - instead, journal.getEntries should be used, as it is much faster.
+func filterExact(dn DN) filter {
+ return func(origin DN, _ Severity) bool {
+ return origin == dn
+ }
+}
+
+// filterSubtree returns a filter that accepts all log entries at a given DN and sub-DNs. For example, filterSubtree at
+// "foo.bar" would allow entries at "foo.bar", "foo.bar.baz", but not "foo" or "foo.barr".
+func filterSubtree(root DN) filter {
+ if root == "" {
+ return filterAll()
+ }
+
+ rootParts := strings.Split(string(root), ".")
+ return func(origin DN, _ Severity) bool {
+ parts := strings.Split(string(origin), ".")
+ if len(parts) < len(rootParts) {
+ return false
+ }
+
+ for i, p := range rootParts {
+ if parts[i] != p {
+ return false
+ }
+ }
+
+ return true
+ }
+}
+
+// filterSeverity returns a filter that accepts log entries at a given severity level or above. See the Severity type
+// for more information about severity levels.
+func filterSeverity(atLeast Severity) filter {
+ return func(origin DN, s Severity) bool {
+ return s.AtLeast(atLeast)
+ }
+}
+
+// scanEntries does a linear scan through the global entry list and returns all entries that match the given filters. If
+// retrieving entries for an exact event, getEntries should be used instead, as it will leverage DN-local linked lists
+// to retrieve them faster.
+// journal.mu must be taken at R or RW level when calling this function.
+func (j *journal) scanEntries(filters ...filter) (res []*entry) {
+ cur := j.tail
+ for {
+ if cur == nil {
+ return
+ }
+
+ passed := true
+ for _, filter := range filters {
+ if !filter(cur.origin, cur.payload.severity) {
+ passed = false
+ break
+ }
+ }
+ if passed {
+ res = append(res, cur)
+ }
+ cur = cur.nextGlobal
+ }
+}
+
+// getEntries returns all entries at a given DN. This is faster then a scanEntries(filterExact), as it uses the special
+// local linked list pointers to traverse the journal. Additional filters can be passed to further limit the entries
+// returned, but a scan through this DN's local linked list will be performed regardless.
+// journal.mu must be taken at R or RW level when calling this function.
+func (j *journal) getEntries(exact DN, filters ...filter) (res []*entry) {
+ cur := j.tails[exact]
+ for {
+ if cur == nil {
+ return
+ }
+
+ passed := true
+ for _, filter := range filters {
+ if !filter(cur.origin, cur.payload.severity) {
+ passed = false
+ break
+ }
+ }
+ if passed {
+ res = append(res, cur)
+ }
+ cur = cur.nextLocal
+ }
+
+}
diff --git a/core/pkg/logtree/journal_entry.go b/core/pkg/logtree/journal_entry.go
new file mode 100644
index 0000000..10a9ea4
--- /dev/null
+++ b/core/pkg/logtree/journal_entry.go
@@ -0,0 +1,154 @@
+// Copyright 2020 The Monogon Project Authors.
+//
+// SPDX-License-Identifier: Apache-2.0
+//
+// Licensed under the Apache License, Version 2.0 (the "License");
+// you may not use this file except in compliance with the License.
+// You may obtain a copy of the License at
+//
+// http://www.apache.org/licenses/LICENSE-2.0
+//
+// Unless required by applicable law or agreed to in writing, software
+// distributed under the License is distributed on an "AS IS" BASIS,
+// WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+// See the License for the specific language governing permissions and
+// limitations under the License.
+
+package logtree
+
+// entry is a journal entry, representing a single log event (encompassed in a Payload) at a given DN.
+// See the journal struct for more information about the global/local linked lists.
+type entry struct {
+ // origin is the DN at which the log entry was recorded, or conversely, in which DN it will be available at.
+ origin DN
+ // journal is the parent journal of this entry. An entry can belong only to a single journal. This pointer is used
+ // to mutate the journal's head/tail pointers when unlinking an entry.
+ journal *journal
+ // payload is the inner log entry Payload. It contains all data and metadata received from the log producer.
+ payload *Payload
+
+ // prevGlobal is the previous entry in the global linked list, or nil if this entry is the oldest entry in the
+ // global linked list.
+ prevGlobal *entry
+ // nextGlobal is the next entry in the global linked list, or nil if this entry is the newest entry in the global
+ // linked list.
+ nextGlobal *entry
+
+ // prevLocal is the previous entry in this entry DN's local linked list, or nil if this entry is the oldest entry in
+ // this local linked list.
+ prevLocal *entry
+ // prevLocal is the next entry in this entry DN's local linked list, or nil if this entry is the newest entry in
+ // this local linked list.
+ nextLocal *entry
+
+ // seqLocal is a counter within a local linked list that increases by one each time a new log entry is added. It is
+ // used to quickly establish local linked list sizes (by subtracting seqLocal from both ends). This setup allows for
+ // O(1) length calculation for local linked lists as long as entries are only unlinked from the head or tail (which
+ // is the case in the current implementation).
+ seqLocal uint64
+}
+
+// unlink removes this entry from both global and local linked lists, updating the journal's head/tail pointers if
+// needed.
+// journal.mu must be taken as RW
+func (e *entry) unlink() {
+ // Unlink from the global linked list.
+ if e.prevGlobal != nil {
+ e.prevGlobal.nextGlobal = e.nextGlobal
+ }
+ if e.nextGlobal != nil {
+ e.nextGlobal.prevGlobal = e.prevGlobal
+ }
+ // Update journal head/tail pointers.
+ if e.journal.head == e {
+ e.journal.head = e.prevGlobal
+ }
+ if e.journal.tail == e {
+ e.journal.tail = e.nextGlobal
+ }
+
+ // Unlink from the local linked list.
+ if e.prevLocal != nil {
+ e.prevLocal.nextLocal = e.nextLocal
+ }
+ if e.nextLocal != nil {
+ e.nextLocal.prevLocal = e.prevLocal
+ }
+ // Update journal head/tail pointers.
+ if e.journal.heads[e.origin] == e {
+ e.journal.heads[e.origin] = e.prevLocal
+ }
+ if e.journal.tails[e.origin] == e {
+ e.journal.tails[e.origin] = e.nextLocal
+ }
+}
+
+// quota describes the quota policy for logging at a given DN.
+type quota struct {
+ // origin is the exact DN that this quota applies to.
+ origin DN
+ // max is the maximum count of log entries permitted for this DN - ie, the maximum size of the local linked list.
+ max uint64
+}
+
+// append adds an entry at the head of the global and local linked lists.
+func (j *journal) append(e *entry) {
+ j.mu.Lock()
+ defer j.mu.Unlock()
+
+ e.journal = j
+
+ // Insert at head in global linked list, set pointers.
+ e.nextGlobal = nil
+ e.prevGlobal = j.head
+ if j.head != nil {
+ j.head.nextGlobal = e
+ }
+ j.head = e
+ if j.tail == nil {
+ j.tail = e
+ }
+
+ // Create quota if necessary.
+ if _, ok := j.quota[e.origin]; !ok {
+ j.quota[e.origin] = "a{origin: e.origin, max: 8192}
+ }
+
+ // Insert at head in local linked list, calculate seqLocal, set pointers.
+ e.nextLocal = nil
+ e.prevLocal = j.heads[e.origin]
+ if j.heads[e.origin] != nil {
+ j.heads[e.origin].nextLocal = e
+ e.seqLocal = e.prevLocal.seqLocal + 1
+ } else {
+ e.seqLocal = 0
+ }
+ j.heads[e.origin] = e
+ if j.tails[e.origin] == nil {
+ j.tails[e.origin] = e
+ }
+
+ // Apply quota to the local linked list that this entry got inserted to, ie. remove elements in excess of the
+ // quota.max count.
+ quota := j.quota[e.origin]
+ count := (j.heads[e.origin].seqLocal - j.tails[e.origin].seqLocal) + 1
+ if count > quota.max {
+ // Keep popping elements off the tail of the local linked list until quota is not violated.
+ left := count - quota.max
+ cur := j.tails[e.origin]
+ for {
+ // This shouldn't happen if quota.max >= 1.
+ if cur == nil {
+ break
+ }
+ if left == 0 {
+ break
+ }
+ el := cur
+ cur = el.nextLocal
+ // Unlinking the entry unlinks it from both the global and local linked lists.
+ el.unlink()
+ left -= 1
+ }
+ }
+}
diff --git a/core/pkg/logtree/journal_subscriber.go b/core/pkg/logtree/journal_subscriber.go
new file mode 100644
index 0000000..18b8124
--- /dev/null
+++ b/core/pkg/logtree/journal_subscriber.go
@@ -0,0 +1,69 @@
+// Copyright 2020 The Monogon Project Authors.
+//
+// SPDX-License-Identifier: Apache-2.0
+//
+// Licensed under the Apache License, Version 2.0 (the "License");
+// you may not use this file except in compliance with the License.
+// You may obtain a copy of the License at
+//
+// http://www.apache.org/licenses/LICENSE-2.0
+//
+// Unless required by applicable law or agreed to in writing, software
+// distributed under the License is distributed on an "AS IS" BASIS,
+// WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+// See the License for the specific language governing permissions and
+// limitations under the License.
+
+package logtree
+
+import (
+ "sync/atomic"
+)
+
+// subscriber is an observer for new entries that are appended to the journal.
+type subscriber struct {
+ // filters that entries need to pass through in order to be sent to the subscriber.
+ filters []filter
+ // dataC is the channel to which entries that pass filters will be sent. The channel must be drained regularly in
+ // order to prevent accumulation of goroutines and possible reordering of messages.
+ dataC chan *LogEntry
+ // doneC is a channel that is closed once the subscriber wishes to stop receiving notifications.
+ doneC chan struct{}
+ // missed is the amount of messages missed by the subscriber by not receiving from dataC fast enough
+ missed uint64
+}
+
+// subscribe attaches a subscriber to the journal.
+// mu must be taken in W mode
+func (j *journal) subscribe(sub *subscriber) {
+ j.subscribers = append(j.subscribers, sub)
+}
+
+// notify sends an entry to all subscribers that wish to receive it.
+func (j *journal) notify(e *entry) {
+ j.mu.Lock()
+ defer j.mu.Unlock()
+
+ newSub := make([]*subscriber, 0, len(j.subscribers))
+ for _, sub := range j.subscribers {
+ select {
+ case <-sub.doneC:
+ close(sub.dataC)
+ continue
+ default:
+ newSub = append(newSub, sub)
+ }
+
+ for _, filter := range sub.filters {
+ if !filter(e.origin, e.payload.severity) {
+ continue
+ }
+ }
+ select {
+ case sub.dataC <- &LogEntry{Payload: e.payload, DN: e.origin}:
+ default:
+ atomic.AddUint64(&sub.missed, 1)
+ }
+ }
+ j.subscribers = newSub
+}
diff --git a/core/pkg/logtree/journal_test.go b/core/pkg/logtree/journal_test.go
new file mode 100644
index 0000000..1a2f8ec
--- /dev/null
+++ b/core/pkg/logtree/journal_test.go
@@ -0,0 +1,136 @@
+// Copyright 2020 The Monogon Project Authors.
+//
+// SPDX-License-Identifier: Apache-2.0
+//
+// Licensed under the Apache License, Version 2.0 (the "License");
+// you may not use this file except in compliance with the License.
+// You may obtain a copy of the License at
+//
+// http://www.apache.org/licenses/LICENSE-2.0
+//
+// Unless required by applicable law or agreed to in writing, software
+// distributed under the License is distributed on an "AS IS" BASIS,
+// WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+// See the License for the specific language governing permissions and
+// limitations under the License.
+
+package logtree
+
+import (
+ "fmt"
+ "testing"
+)
+
+func TestJournalRetention(t *testing.T) {
+ j := newJournal()
+
+ for i := 0; i < 9000; i += 1 {
+ e := &entry{
+ origin: "main",
+ payload: testPayload(fmt.Sprintf("test %d", i)),
+ }
+ j.append(e)
+ }
+
+ entries := j.getEntries("main")
+ if want, got := 8192, len(entries); want != got {
+ t.Fatalf("wanted %d entries, got %d", want, got)
+ }
+ for i, entry := range entries {
+ want := fmt.Sprintf("test %d", (9000-8192)+i)
+ got := entry.payload.message
+ if want != got {
+ t.Fatalf("wanted entry %q, got %q", want, got)
+ }
+ }
+}
+
+func TestJournalQuota(t *testing.T) {
+ j := newJournal()
+
+ for i := 0; i < 9000; i += 1 {
+ j.append(&entry{
+ origin: "chatty",
+ payload: testPayload(fmt.Sprintf("chatty %d", i)),
+ })
+ if i%10 == 0 {
+ j.append(&entry{
+ origin: "solemn",
+ payload: testPayload(fmt.Sprintf("solemn %d", i)),
+ })
+ }
+ }
+
+ entries := j.getEntries("chatty")
+ if want, got := 8192, len(entries); want != got {
+ t.Fatalf("wanted %d chatty entries, got %d", want, got)
+ }
+ entries = j.getEntries("solemn")
+ if want, got := 900, len(entries); want != got {
+ t.Fatalf("wanted %d solemn entries, got %d", want, got)
+ }
+ entries = j.getEntries("absent")
+ if want, got := 0, len(entries); want != got {
+ t.Fatalf("wanted %d absent entries, got %d", want, got)
+ }
+
+ entries = j.scanEntries(filterAll())
+ if want, got := 8192+900, len(entries); want != got {
+ t.Fatalf("wanted %d total entries, got %d", want, got)
+ }
+ setMessages := make(map[string]bool)
+ for _, entry := range entries {
+ setMessages[entry.payload.message] = true
+ }
+
+ for i := 0; i < 900; i += 1 {
+ want := fmt.Sprintf("solemn %d", i*10)
+ if !setMessages[want] {
+ t.Fatalf("could not find entry %q in journal", want)
+ }
+ }
+ for i := 0; i < 8192; i += 1 {
+ want := fmt.Sprintf("chatty %d", i+(9000-8192))
+ if !setMessages[want] {
+ t.Fatalf("could not find entry %q in journal", want)
+ }
+ }
+}
+
+func TestJournalSubtree(t *testing.T) {
+ j := newJournal()
+ j.append(&entry{origin: "a", payload: testPayload("a")})
+ j.append(&entry{origin: "a.b", payload: testPayload("a.b")})
+ j.append(&entry{origin: "a.b.c", payload: testPayload("a.b.c")})
+ j.append(&entry{origin: "a.b.d", payload: testPayload("a.b.d")})
+ j.append(&entry{origin: "e.f", payload: testPayload("e.f")})
+ j.append(&entry{origin: "e.g", payload: testPayload("e.g")})
+
+ expect := func(f filter, msgs ...string) string {
+ res := j.scanEntries(f)
+ set := make(map[string]bool)
+ for _, entry := range res {
+ set[entry.payload.message] = true
+ }
+
+ for _, want := range msgs {
+ if !set[want] {
+ return fmt.Sprintf("missing entry %q", want)
+ }
+ }
+ return ""
+ }
+
+ if res := expect(filterAll(), "a", "a.b", "a.b.c", "a.b.d", "e.f", "e.g"); res != "" {
+ t.Fatalf("All: %s", res)
+ }
+ if res := expect(filterSubtree("a"), "a", "a.b", "a.b.c", "a.b.d"); res != "" {
+ t.Fatalf("Subtree(a): %s", res)
+ }
+ if res := expect(filterSubtree("a.b"), "a.b", "a.b.c", "a.b.d"); res != "" {
+ t.Fatalf("Subtree(a.b): %s", res)
+ }
+ if res := expect(filterSubtree("e"), "e.f", "e.g"); res != "" {
+ t.Fatalf("Subtree(a.b): %s", res)
+ }
+}
diff --git a/core/pkg/logtree/leveled.go b/core/pkg/logtree/leveled.go
index 7e93d24..2c8fcc4 100644
--- a/core/pkg/logtree/leveled.go
+++ b/core/pkg/logtree/leveled.go
@@ -76,3 +76,33 @@
// Infof is the equivalent of a LeveledLogger's Infof call, guarded by whether this VerboseLeveledLogger is enabled.
Infof(format string, args ...interface{})
}
+
+// Severity is one of the severities as described in LeveledLogger.
+type Severity string
+
+const (
+ INFO Severity = "I"
+ WARNING Severity = "W"
+ ERROR Severity = "E"
+ FATAL Severity = "F"
+)
+
+var (
+ // SeverityAtLeast maps a given severity to a list of severities that at that severity or higher. In other words,
+ // SeverityAtLeast[X] returns a list of severities that might be seen in a log at severity X.
+ SeverityAtLeast = map[Severity][]Severity{
+ INFO: {INFO, WARNING, ERROR, FATAL},
+ WARNING: {WARNING, ERROR, FATAL},
+ ERROR: {ERROR, FATAL},
+ FATAL: {FATAL},
+ }
+)
+
+func (s Severity) AtLeast(other Severity) bool {
+ for _, el := range SeverityAtLeast[other] {
+ if el == s {
+ return true
+ }
+ }
+ return false
+}
diff --git a/core/pkg/logtree/logtree.go b/core/pkg/logtree/logtree.go
new file mode 100644
index 0000000..4d674fd
--- /dev/null
+++ b/core/pkg/logtree/logtree.go
@@ -0,0 +1,141 @@
+// Copyright 2020 The Monogon Project Authors.
+//
+// SPDX-License-Identifier: Apache-2.0
+//
+// Licensed under the Apache License, Version 2.0 (the "License");
+// you may not use this file except in compliance with the License.
+// You may obtain a copy of the License at
+//
+// http://www.apache.org/licenses/LICENSE-2.0
+//
+// Unless required by applicable law or agreed to in writing, software
+// distributed under the License is distributed on an "AS IS" BASIS,
+// WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+// See the License for the specific language governing permissions and
+// limitations under the License.
+
+package logtree
+
+import (
+ "fmt"
+ "strings"
+ "sync"
+)
+
+// LogTree is a tree-shapped logging system. For more information, see the package-level documentation.
+type LogTree struct {
+ // journal is the tree's journal, storing all log data and managing subscribers.
+ journal *journal
+ // root is the root node of the actual tree of the log tree. The nodes contain per-DN configuration options, notably
+ // the current verbosity level of that DN.
+ root *node
+}
+
+func New() *LogTree {
+ lt := &LogTree{
+ journal: newJournal(),
+ }
+ lt.root = newNode(lt, "")
+ return lt
+}
+
+// node represents a given DN as a discrete 'logger'. It implementes the LeveledLogger interface for log publishing,
+// entries from which it passes over to the logtree's journal.
+type node struct {
+ // dn is the DN which this node represents (or "" if this is the root node).
+ dn DN
+ // tree is the LogTree to which this node belongs.
+ tree *LogTree
+ // verbosity is the current verbosity level of this DN/node, affecting .V(n) LeveledLogger calls
+ verbosity VerbosityLevel
+
+ // mu guards children.
+ mu sync.Mutex
+ // children is a map of DN-part to a children node in the logtree. A DN-part is a string representing a part of the
+ // DN between the deliming dots, as returned by DN.Path.
+ children map[string]*node
+}
+
+// newNode returns a node at a given DN in the LogTree - but doesn't set up the LogTree to insert it accordingly.
+func newNode(tree *LogTree, dn DN) *node {
+ n := &node{
+ dn: dn,
+ tree: tree,
+ children: make(map[string]*node),
+ }
+ return n
+}
+
+// nodeByDN returns the LogTree node corresponding to a given DN. If either the node or some of its parents do not
+// exist they will be created as needed.
+func (l *LogTree) nodeByDN(dn DN) (*node, error) {
+ traversal, err := newTraversal(dn)
+ if err != nil {
+ return nil, fmt.Errorf("traversal failed: %w", err)
+ }
+ return traversal.execute(l.root), nil
+}
+
+// nodeTraversal represents a request to traverse the LogTree in search of a given node by DN.
+type nodeTraversal struct {
+ // want is the DN of the node's that requested to be found.
+ want DN
+ // current is the path already taken to find the node, in the form of DN parts. It starts out as want.Parts() and
+ // progresses to become empty as the traversal continues.
+ current []string
+ // left is the path that's still needed to be taken in order to find the node, in the form of DN parts. It starts
+ // out empty and progresses to become wants.Parts() as the traversal continues.
+ left []string
+}
+
+// next adjusts the traversal's current/left slices to the next element of the traversal, returns the part that's now
+// being looked for (or "" if the traveral is done) and the full DN of the element that's being looked for.
+//
+// For example, a traversal of foo.bar.baz will cause .next() to return the following on each invocation:
+// - part: foo, full: foo
+// - part: bar, full: foo.bar
+// - part: baz, full: foo.bar.baz
+// - part: "", full: foo.bar.baz
+func (t *nodeTraversal) next() (part string, full DN) {
+ if len(t.left) == 0 {
+ return "", t.want
+ }
+ part = t.left[0]
+ t.current = append(t.current, part)
+ t.left = t.left[1:]
+ full = DN(strings.Join(t.current, "."))
+ return
+}
+
+// newTraversal returns a nodeTraversal fora a given wanted DN.
+func newTraversal(dn DN) (*nodeTraversal, error) {
+ parts, err := dn.Path()
+ if err != nil {
+ return nil, err
+ }
+ return &nodeTraversal{
+ want: dn,
+ left: parts,
+ }, nil
+}
+
+// execute the traversal in order to find the node. This can only be called once per traversal.
+// Nodes will be created within the tree until the target node is reached. Existing nodes will be reused.
+// This is effectively an idempotent way of accessing a node in the tree based on a traversal.
+func (t *nodeTraversal) execute(n *node) *node {
+ cur := n
+ for {
+ part, full := t.next()
+ if part == "" {
+ return cur
+ }
+
+ mu := &cur.mu
+ mu.Lock()
+ if _, ok := cur.children[part]; !ok {
+ cur.children[part] = newNode(n.tree, DN(full))
+ }
+ cur = cur.children[part]
+ mu.Unlock()
+ }
+}
diff --git a/core/pkg/logtree/logtree_access.go b/core/pkg/logtree/logtree_access.go
new file mode 100644
index 0000000..045f105
--- /dev/null
+++ b/core/pkg/logtree/logtree_access.go
@@ -0,0 +1,157 @@
+// Copyright 2020 The Monogon Project Authors.
+//
+// SPDX-License-Identifier: Apache-2.0
+//
+// Licensed under the Apache License, Version 2.0 (the "License");
+// you may not use this file except in compliance with the License.
+// You may obtain a copy of the License at
+//
+// http://www.apache.org/licenses/LICENSE-2.0
+//
+// Unless required by applicable law or agreed to in writing, software
+// distributed under the License is distributed on an "AS IS" BASIS,
+// WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+// See the License for the specific language governing permissions and
+// limitations under the License.
+
+package logtree
+
+import "sync/atomic"
+
+// LogReadOption describes options for the LogTree.Read call.
+type LogReadOption struct {
+ withChildren bool
+ withStream bool
+ withBacklog int
+ withMinimumSeverity Severity
+}
+
+// WithChildren makes Read return/stream data for both a given DN and all its children.
+func WithChildren() LogReadOption { return LogReadOption{withChildren: true} }
+
+// WithStream makes Read return a stream of data. This works alongside WithBacklog to create a read-and-stream
+// construct.
+func WithStream() LogReadOption { return LogReadOption{withStream: true} }
+
+// WithBacklog makes Read return already recorded log entries, up to count elements.
+func WithBacklog(count int) LogReadOption { return LogReadOption{withBacklog: count} }
+
+// BacklogAllAvailable makes WithBacklog return all backlogged log data that logtree possesses.
+const BacklogAllAvailable int = -1
+
+// WithMinimumSeverity makes Read return only log entries that are at least at a given Severity.
+func WithMinimumSeverity(s Severity) LogReadOption {
+ return LogReadOption{withMinimumSeverity: s}
+}
+
+// LogReader permits reading an already existing backlog of log entries and to stream further ones.
+type LogReader struct {
+ // Backlog are the log entries already logged by LogTree. This will only be set if WithBacklog has been passed to
+ // Read.
+ Backlog []*LogEntry
+ // Stream is a channel of new entries as received live by LogTree. This will only be set if WithStream has been
+ // passed to Read. In this case, entries from this channel must be read as fast as possible by the consumer in order
+ // to prevent missing entries.
+ Stream <-chan *LogEntry
+ // done is channel used to signal (by closing) that the log consumer is not interested in more Stream data.
+ done chan<- struct{}
+ // missed is an atomic integer pointer that tells the subscriber how many messages in Stream they missed. This
+ // pointer is nil if no streaming has been requested.
+ missed *uint64
+}
+
+// Missed returns the amount of entries that were missed from Stream (as the channel was not drained fast enough).
+func (l *LogReader) Missed() uint64 {
+ // No Stream.
+ if l.missed == nil {
+ return 0
+ }
+ return atomic.LoadUint64(l.missed)
+}
+
+// Close closes the LogReader's Stream. This must be called once the Reader does not wish to receive streaming messages
+// anymore.
+func (l *LogReader) Close() {
+ if l.done != nil {
+ close(l.done)
+ }
+}
+
+type LogEntry struct {
+ *Payload
+ DN DN
+}
+
+// Read and/or stream entries from a LogTree. The returned LogReader is influenced by the LogReadOptions passed, which
+// influence whether the Read will return existing entries, a stream, or both. In addition the options also dictate
+// whether only entries for that particular DN are returned, or for all sub-DNs as well.
+func (l *LogTree) Read(dn DN, opts ...LogReadOption) *LogReader {
+ l.journal.mu.RLock()
+ defer l.journal.mu.RUnlock()
+
+ var backlog int
+ var stream bool
+ var recursive bool
+ var severity Severity
+
+ for _, opt := range opts {
+ if opt.withBacklog > 0 || opt.withBacklog == BacklogAllAvailable {
+ backlog = opt.withBacklog
+ }
+ if opt.withStream {
+ stream = true
+ }
+ if opt.withChildren {
+ recursive = true
+ }
+ if opt.withMinimumSeverity != "" {
+ severity = opt.withMinimumSeverity
+ }
+ }
+
+ var filters []filter
+ if recursive {
+ filters = append(filters, filterSubtree(dn))
+ } else {
+ filters = append(filters, filterExact(dn))
+ }
+ if severity != "" {
+ filters = append(filters, filterSeverity(severity))
+ }
+
+ var entries []*entry
+ if backlog > 0 || backlog == BacklogAllAvailable {
+ // TODO(q3k): pass over the backlog count to scanEntries/getEntries, instead of discarding them here.
+ if recursive {
+ entries = l.journal.scanEntries(filters...)
+ } else {
+ entries = l.journal.getEntries(dn, filters...)
+ }
+ if backlog != BacklogAllAvailable && len(entries) > backlog {
+ entries = entries[:backlog]
+ }
+ }
+
+ var sub *subscriber
+ if stream {
+ sub = &subscriber{
+ // TODO(q3k): make buffer size configurable
+ dataC: make(chan *LogEntry, 128),
+ doneC: make(chan struct{}),
+ filters: filters,
+ }
+ l.journal.subscribe(sub)
+ }
+
+ lr := &LogReader{}
+ lr.Backlog = make([]*LogEntry, len(entries))
+ for i, entry := range entries {
+ lr.Backlog[i] = &LogEntry{Payload: entry.payload, DN: entry.origin}
+ }
+ if stream {
+ lr.Stream = sub.dataC
+ lr.done = sub.doneC
+ lr.missed = &sub.missed
+ }
+ return lr
+}
diff --git a/core/pkg/logtree/logtree_publisher.go b/core/pkg/logtree/logtree_publisher.go
new file mode 100644
index 0000000..02243ac
--- /dev/null
+++ b/core/pkg/logtree/logtree_publisher.go
@@ -0,0 +1,151 @@
+// Copyright 2020 The Monogon Project Authors.
+//
+// SPDX-License-Identifier: Apache-2.0
+//
+// Licensed under the Apache License, Version 2.0 (the "License");
+// you may not use this file except in compliance with the License.
+// You may obtain a copy of the License at
+//
+// http://www.apache.org/licenses/LICENSE-2.0
+//
+// Unless required by applicable law or agreed to in writing, software
+// distributed under the License is distributed on an "AS IS" BASIS,
+// WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+// See the License for the specific language governing permissions and
+// limitations under the License.
+
+package logtree
+
+import (
+ "fmt"
+ "runtime"
+ "strings"
+ "time"
+)
+
+// LeveledFor returns a LeveledLogger publishing interface for a given DN. An error may be returned if the DN is
+// malformed.
+func (l *LogTree) LeveledFor(dn DN) (LeveledLogger, error) {
+ return l.nodeByDN(dn)
+}
+
+// MustLeveledFor returns a LeveledLogger publishing interface for a given DN, or panics if the given DN is invalid.
+func (l *LogTree) MustLeveledFor(dn DN) LeveledLogger {
+ leveled, err := l.LeveledFor(dn)
+ if err != nil {
+ panic(fmt.Errorf("LeveledFor returned: %w", err))
+ }
+ return leveled
+}
+
+// SetVerbosity sets the verbosity for a given DN (non-recursively, ie. for that DN only, not its children).
+func (l *LogTree) SetVerbosity(dn DN, level VerbosityLevel) error {
+ node, err := l.nodeByDN(dn)
+ if err != nil {
+ return err
+ }
+ node.verbosity = level
+ return nil
+}
+
+// log builds a Payload and entry for a given message, including all related metadata, and appends it to the journal,
+// notifying all pertinent subscribers.
+func (n *node) log(depth int, severity Severity, msg string) {
+ _, file, line, ok := runtime.Caller(2 + depth)
+ if !ok {
+ file = "???"
+ line = 1
+ } else {
+ slash := strings.LastIndex(file, "/")
+ if slash >= 0 {
+ file = file[slash+1:]
+ }
+ }
+ p := &Payload{
+ timestamp: time.Now(),
+ severity: severity,
+ message: msg,
+ file: file,
+ line: line,
+ }
+ e := &entry{
+ origin: n.dn,
+ payload: p,
+ }
+ n.tree.journal.append(e)
+ n.tree.journal.notify(e)
+}
+
+// Info implements the LeveledLogger interface.
+func (n *node) Info(args ...interface{}) {
+ n.log(0, INFO, fmt.Sprint(args...))
+}
+
+// Infof implements the LeveledLogger interface.
+func (n *node) Infof(format string, args ...interface{}) {
+ n.log(0, INFO, fmt.Sprintf(format, args...))
+}
+
+// Warning implements the LeveledLogger interface.
+func (n *node) Warning(args ...interface{}) {
+ n.log(0, WARNING, fmt.Sprint(args...))
+}
+
+// Warningf implements the LeveledLogger interface.
+func (n *node) Warningf(format string, args ...interface{}) {
+ n.log(0, WARNING, fmt.Sprintf(format, args...))
+}
+
+// Error implements the LeveledLogger interface.
+func (n *node) Error(args ...interface{}) {
+ n.log(0, ERROR, fmt.Sprint(args...))
+}
+
+// Errorf implements the LeveledLogger interface.
+func (n *node) Errorf(format string, args ...interface{}) {
+ n.log(0, ERROR, fmt.Sprintf(format, args...))
+}
+
+// Fatal implements the LeveledLogger interface.
+func (n *node) Fatal(args ...interface{}) {
+ n.log(0, FATAL, fmt.Sprint(args...))
+}
+
+// Fatalf implements the LeveledLogger interface.
+func (n *node) Fatalf(format string, args ...interface{}) {
+ n.log(0, FATAL, fmt.Sprintf(format, args...))
+}
+
+// V implements the LeveledLogger interface.
+func (n *node) V(v VerbosityLevel) VerboseLeveledLogger {
+ return &verbose{
+ node: n,
+ enabled: n.verbosity >= v,
+ }
+}
+
+// verbose implements the VerboseLeveledLogger interface. It is a thin wrapper around node, with an 'enabled' bool. This
+// means that V(n)-returned VerboseLeveledLoggers must be short lived, as a changed in verbosity will not affect all
+// already existing VerboseLeveledLoggers.
+type verbose struct {
+ node *node
+ enabled bool
+}
+
+func (v *verbose) Enabled() bool {
+ return v.enabled
+}
+
+func (v *verbose) Info(args ...interface{}) {
+ if !v.enabled {
+ return
+ }
+ v.node.log(0, INFO, fmt.Sprint(args...))
+}
+
+func (v *verbose) Infof(format string, args ...interface{}) {
+ if !v.enabled {
+ return
+ }
+ v.node.log(0, INFO, fmt.Sprintf(format, args...))
+}
diff --git a/core/pkg/logtree/logtree_test.go b/core/pkg/logtree/logtree_test.go
new file mode 100644
index 0000000..3e5dfba
--- /dev/null
+++ b/core/pkg/logtree/logtree_test.go
@@ -0,0 +1,154 @@
+// Copyright 2020 The Monogon Project Authors.
+//
+// SPDX-License-Identifier: Apache-2.0
+//
+// Licensed under the Apache License, Version 2.0 (the "License");
+// you may not use this file except in compliance with the License.
+// You may obtain a copy of the License at
+//
+// http://www.apache.org/licenses/LICENSE-2.0
+//
+// Unless required by applicable law or agreed to in writing, software
+// distributed under the License is distributed on an "AS IS" BASIS,
+// WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+// See the License for the specific language governing permissions and
+// limitations under the License.
+
+package logtree
+
+import (
+ "fmt"
+ "strings"
+ "testing"
+ "time"
+)
+
+func TestBacklog(t *testing.T) {
+ tree := New()
+ tree.MustLeveledFor("main").Info("hello, main!")
+ tree.MustLeveledFor("main.foo").Info("hello, main.foo!")
+ tree.MustLeveledFor("main.bar").Info("hello, main.bar!")
+ tree.MustLeveledFor("aux").Info("hello, aux!")
+
+ expect := func(dn DN, entries ...string) string {
+ res := tree.Read(dn, WithChildren(), WithBacklog(BacklogAllAvailable))
+ if want, got := len(entries), len(res.Backlog); want != got {
+ t.Fatalf("wanted %d backlog entries, got %d", want, got)
+ }
+ got := make(map[string]bool)
+ for _, entry := range res.Backlog {
+ got[entry.Message()] = true
+ }
+ for _, entry := range entries {
+ if !got[entry] {
+ return fmt.Sprintf("missing entry %q", entry)
+ }
+ }
+ return ""
+ }
+
+ if res := expect("main", "hello, main!", "hello, main.foo!", "hello, main.bar!"); res != "" {
+ t.Errorf("retrieval at main failed: %s", res)
+ }
+ if res := expect("", "hello, main!", "hello, main.foo!", "hello, main.bar!", "hello, aux!"); res != "" {
+ t.Errorf("retrieval at root failed: %s", res)
+ }
+ if res := expect("aux", "hello, aux!"); res != "" {
+ t.Errorf("retrieval at aux failed: %s", res)
+ }
+}
+
+func TestStream(t *testing.T) {
+ tree := New()
+ tree.MustLeveledFor("main").Info("hello, backlog")
+
+ res := tree.Read("", WithBacklog(BacklogAllAvailable), WithChildren(), WithStream())
+ defer res.Close()
+ if want, got := 1, len(res.Backlog); want != got {
+ t.Errorf("wanted %d backlog item, got %d", want, got)
+ }
+
+ tree.MustLeveledFor("main").Info("hello, stream")
+
+ select {
+ case <-time.After(time.Second * 1):
+ t.Fatalf("timeout elapsed")
+ case p := <-res.Stream:
+ if want, got := "hello, stream", p.Message(); want != got {
+ t.Fatalf("stream returned %q, wanted %q", got, want)
+ }
+ }
+}
+
+func TestVerbose(t *testing.T) {
+ tree := New()
+
+ tree.MustLeveledFor("main").V(10).Info("this shouldn't get logged")
+
+ reader := tree.Read("", WithBacklog(BacklogAllAvailable), WithChildren())
+ if want, got := 0, len(reader.Backlog); want != got {
+ t.Fatalf("expected nothing to be logged, got %+v", reader.Backlog)
+ }
+
+ tree.SetVerbosity("main", 10)
+ tree.MustLeveledFor("main").V(10).Info("this should get logged")
+
+ reader = tree.Read("", WithBacklog(BacklogAllAvailable), WithChildren())
+ if want, got := 1, len(reader.Backlog); want != got {
+ t.Fatalf("expected %d entries to get logged, got %d", want, got)
+ }
+}
+
+func TestMetadata(t *testing.T) {
+ tree := New()
+ tree.MustLeveledFor("main").Error("i am an error")
+ tree.MustLeveledFor("main").Warning("i am a warning")
+ tree.MustLeveledFor("main").Info("i am informative")
+ tree.MustLeveledFor("main").V(0).Info("i am a zero-level debug")
+
+ reader := tree.Read("", WithChildren(), WithBacklog(BacklogAllAvailable))
+ if want, got := 4, len(reader.Backlog); want != got {
+ t.Fatalf("expected %d entries, got %d", want, got)
+ }
+
+ for _, te := range []struct {
+ ix int
+ severity Severity
+ message string
+ }{
+ {0, ERROR, "i am an error"},
+ {1, WARNING, "i am a warning"},
+ {2, INFO, "i am informative"},
+ {3, INFO, "i am a zero-level debug"},
+ } {
+ p := reader.Backlog[te.ix]
+ if want, got := te.severity, p.Severity(); want != got {
+ t.Errorf("wanted element %d to have severity %s, got %s", te.ix, want, got)
+ }
+ if want, got := te.message, p.Message(); want != got {
+ t.Errorf("wanted element %d to have message %q, got %q", te.ix, want, got)
+ }
+ if want, got := "logtree_test.go", strings.Split(p.Location(), ":")[0]; want != got {
+ t.Errorf("wanted element %d to have file %q, got %q", te.ix, want, got)
+ }
+ }
+}
+
+func TestSeverity(t *testing.T) {
+ tree := New()
+ tree.MustLeveledFor("main").Error("i am an error")
+ tree.MustLeveledFor("main").Warning("i am a warning")
+ tree.MustLeveledFor("main").Info("i am informative")
+ tree.MustLeveledFor("main").V(0).Info("i am a zero-level debug")
+
+ reader := tree.Read("main", WithBacklog(BacklogAllAvailable), WithMinimumSeverity(WARNING))
+ if want, got := 2, len(reader.Backlog); want != got {
+ t.Fatalf("wanted %d entries, got %d", want, got)
+ }
+ if want, got := "i am an error", reader.Backlog[0].Message(); want != got {
+ t.Fatalf("wanted entry %q, got %q", want, got)
+ }
+ if want, got := "i am a warning", reader.Backlog[1].Message(); want != got {
+ t.Fatalf("wanted entry %q, got %q", want, got)
+ }
+}
diff --git a/core/pkg/logtree/payload.go b/core/pkg/logtree/payload.go
new file mode 100644
index 0000000..c6707e9
--- /dev/null
+++ b/core/pkg/logtree/payload.go
@@ -0,0 +1,64 @@
+// Copyright 2020 The Monogon Project Authors.
+//
+// SPDX-License-Identifier: Apache-2.0
+//
+// Licensed under the Apache License, Version 2.0 (the "License");
+// you may not use this file except in compliance with the License.
+// You may obtain a copy of the License at
+//
+// http://www.apache.org/licenses/LICENSE-2.0
+//
+// Unless required by applicable law or agreed to in writing, software
+// distributed under the License is distributed on an "AS IS" BASIS,
+// WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+// See the License for the specific language governing permissions and
+// limitations under the License.
+
+package logtree
+
+import (
+ "fmt"
+ "time"
+)
+
+// Payload is a log entry for leveled logs (as per leveled.go). It contains not only the log message itself and its
+// severity, but also additional metadata that would be usually seen in a text representation of a leveled log entry.
+type Payload struct {
+ // message is the log message, rendered from a leveled log call like Infof(), Warningf(), ...
+ message string
+ // timestamp is the time at which this message was emitted.
+ timestamp time.Time
+ // severity is the leveled Severity at which this message was emitted.
+ severity Severity
+ // file is the filename of the caller that emitted this message.
+ file string
+ // line is the line number within the file of the caller that emitted this message.
+ line int
+}
+
+func (p *Payload) String() string {
+ // Same format as in glog:
+ // Lmmdd hh:mm:ss.uuuuuu threadid file:line]
+ // Except, threadid is (currently) always zero. In the future this field might be used for something different.
+
+ _, month, day := p.timestamp.Date()
+ hour, minute, second := p.timestamp.Clock()
+ nsec := p.timestamp.Nanosecond() / 1000
+
+ // TODO(q3k): rewrite this to printf-less code.
+ return fmt.Sprintf("%s%02d%02d %02d:%02d:%02d.%06d % 7d %s:%d] %s", p.severity, month, day, hour, minute, second,
+ nsec, 0, p.file, p.line, p.message)
+}
+
+// Message returns the inner message of this entry, ie. what was passed to the actual logging method.
+func (p *Payload) Message() string { return p.message }
+
+// Timestamp returns the time at which this entry was logged.
+func (p *Payload) Timestamp() time.Time { return p.timestamp }
+
+// Location returns a string in the form of file_name:line_number that shows the origin of the log entry in the
+// program source.
+func (p *Payload) Location() string { return fmt.Sprintf("%s:%d", p.file, p.line) }
+
+// Severity returns the Severity with which this entry was logged.
+func (p *Payload) Severity() Severity { return p.severity }
diff --git a/core/pkg/logtree/payload_test.go b/core/pkg/logtree/payload_test.go
new file mode 100644
index 0000000..26219cd
--- /dev/null
+++ b/core/pkg/logtree/payload_test.go
@@ -0,0 +1,29 @@
+// Copyright 2020 The Monogon Project Authors.
+//
+// SPDX-License-Identifier: Apache-2.0
+//
+// Licensed under the Apache License, Version 2.0 (the "License");
+// you may not use this file except in compliance with the License.
+// You may obtain a copy of the License at
+//
+// http://www.apache.org/licenses/LICENSE-2.0
+//
+// Unless required by applicable law or agreed to in writing, software
+// distributed under the License is distributed on an "AS IS" BASIS,
+// WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+// See the License for the specific language governing permissions and
+// limitations under the License.
+
+package logtree
+
+import "time"
+
+func testPayload(msg string) *Payload {
+ return &Payload{
+ message: msg,
+ timestamp: time.Now(),
+ severity: INFO,
+ file: "main.go",
+ line: 1337,
+ }
+}