-
Notifications
You must be signed in to change notification settings - Fork 3
Clickhouse exporter improvements for logs #1
New issue
Have a question about this project? Sign up for a free GitHub account to open an issue and contact its maintainers and the community.
By clicking “Sign up for GitHub”, you agree to our terms of service and privacy statement. We’ll occasionally send you account related emails.
Already on GitHub? Sign in to your account
Changes from all commits
bb31c95
1e49cf8
45c9ee2
2d134b6
6f33793
dc74469
678faab
120c6e0
7977dbb
File filter
Filter by extension
Conversations
Jump to
Diff view
Diff view
There are no files selected for viewing
| Original file line number | Diff line number | Diff line change |
|---|---|---|
|
|
@@ -124,7 +124,7 @@ COMMIT?=HEAD | |
| MODSET?=contrib-core | ||
| [email protected]:open-telemetry/opentelemetry-collector-contrib.git | ||
| .PHONY: push-tags | ||
| push-tags: $(MULITMOD) | ||
| push-tags: $(MULITMOD) | ||
| $(MULITMOD) verify | ||
| set -e; for tag in `$(MULITMOD) tag -m ${MODSET} -c ${COMMIT} --print-tags | grep -v "Using" `; do \ | ||
| echo "pushing tag $${tag}"; \ | ||
|
|
||
Some generated files are not rendered by default. Learn more about how customized files appear on GitHub.
| Original file line number | Diff line number | Diff line change |
|---|---|---|
|
|
@@ -18,9 +18,11 @@ import ( | |
| "context" | ||
| "database/sql" | ||
| "fmt" | ||
| "net/url" | ||
| "strings" | ||
| "time" | ||
|
|
||
| _ "github.com/ClickHouse/clickhouse-go/v2" // For register database driver. | ||
| "github.com/ClickHouse/clickhouse-go/v2" | ||
| "go.opentelemetry.io/collector/component" | ||
| "go.opentelemetry.io/collector/pdata/pcommon" | ||
| "go.opentelemetry.io/collector/pdata/plog" | ||
|
|
@@ -39,7 +41,7 @@ type logsExporter struct { | |
| } | ||
|
|
||
| func newLogsExporter(logger *zap.Logger, cfg *Config) (*logsExporter, error) { | ||
| client, err := newClickhouseClient(cfg) | ||
| client, err := newClickHouseConn(cfg) | ||
| if err != nil { | ||
| return nil, err | ||
| } | ||
|
|
@@ -73,28 +75,40 @@ func (e *logsExporter) shutdown(_ context.Context) error { | |
|
|
||
| func (e *logsExporter) pushLogsData(ctx context.Context, ld plog.Logs) error { | ||
| start := time.Now() | ||
| err := doWithTx(ctx, e.client, func(tx *sql.Tx) error { | ||
| statement, err := tx.PrepareContext(ctx, e.insertSQL) | ||
| err := func() error { | ||
| scope, err := e.client.Begin() | ||
|
Collaborator
Author
There was a problem hiding this comment. Choose a reason for hiding this commentThe reason will be displayed to describe this comment to others. Learn more. this is apparently the way to do batch inserts using the database/sql interface of the clickhouse driver https://clickhouse.com/docs/en/integrations/go/clickhouse-go/database-sql-api#batch-insert |
||
| if err != nil { | ||
| return fmt.Errorf("PrepareContext:%w", err) | ||
| return fmt.Errorf("Begin:%w", err) | ||
| } | ||
| defer func() { | ||
| _ = statement.Close() | ||
| }() | ||
|
|
||
| batch, err := scope.Prepare(e.insertSQL) | ||
| if err != nil { | ||
| return fmt.Errorf("Prepare:%w", err) | ||
| } | ||
|
|
||
| var serviceName string | ||
| for i := 0; i < ld.ResourceLogs().Len(); i++ { | ||
| logs := ld.ResourceLogs().At(i) | ||
| resAttr := make(map[string]string) | ||
|
|
||
| resourceLogs := ld.ResourceLogs() | ||
| for i := 0; i < resourceLogs.Len(); i++ { | ||
| logs := resourceLogs.At(i) | ||
| res := logs.Resource() | ||
| resAttr := attributesToMap(res.Attributes()) | ||
| if v, ok := res.Attributes().Get(conventions.AttributeServiceName); ok { | ||
|
|
||
| attrs := res.Attributes() | ||
| attributesToMap(attrs, resAttr) | ||
|
|
||
| if v, ok := attrs.Get(conventions.AttributeServiceName); ok { | ||
| serviceName = v.Str() | ||
| } | ||
| for j := 0; j < logs.ScopeLogs().Len(); j++ { | ||
| rs := logs.ScopeLogs().At(j).LogRecords() | ||
| for k := 0; k < rs.Len(); k++ { | ||
| r := rs.At(k) | ||
| logAttr := attributesToMap(r.Attributes()) | ||
| _, err = statement.ExecContext(ctx, | ||
|
|
||
| logAttr := make(map[string]string, attrs.Len()) | ||
| attributesToMap(r.Attributes(), logAttr) | ||
|
|
||
| _, err = batch.Exec( | ||
| r.Timestamp().AsTime(), | ||
| traceutil.TraceIDToHexOrEmptyString(r.TraceID()), | ||
| traceutil.SpanIDToHexOrEmptyString(r.SpanID()), | ||
|
|
@@ -107,26 +121,31 @@ func (e *logsExporter) pushLogsData(ctx context.Context, ld plog.Logs) error { | |
| logAttr, | ||
| ) | ||
| if err != nil { | ||
| return fmt.Errorf("ExecContext:%w", err) | ||
| return fmt.Errorf("Append:%w", err) | ||
| } | ||
| } | ||
| } | ||
|
|
||
| // clear map for reuse | ||
| for k := range resAttr { | ||
| delete(resAttr, k) | ||
| } | ||
| } | ||
| return nil | ||
| }) | ||
|
|
||
| return scope.Commit() | ||
| }() | ||
|
|
||
| duration := time.Since(start) | ||
| e.logger.Info("insert logs", zap.Int("records", ld.LogRecordCount()), | ||
| zap.String("cost", duration.String())) | ||
| return err | ||
| } | ||
|
|
||
| func attributesToMap(attributes pcommon.Map) map[string]string { | ||
| m := make(map[string]string, attributes.Len()) | ||
| func attributesToMap(attributes pcommon.Map, dest map[string]string) { | ||
| attributes.Range(func(k string, v pcommon.Value) bool { | ||
| m[k] = v.AsString() | ||
| dest[k] = v.AsString() | ||
| return true | ||
| }) | ||
| return m | ||
| } | ||
|
|
||
| const ( | ||
|
|
@@ -155,6 +174,7 @@ PARTITION BY toDate(Timestamp) | |
| ORDER BY (ServiceName, SeverityText, toUnixTimestamp(Timestamp), TraceId) | ||
| SETTINGS index_granularity=8192, ttl_only_drop_parts = 1; | ||
| ` | ||
|
|
||
| // language=ClickHouse SQL | ||
| insertLogsSQLTemplate = `INSERT INTO %s ( | ||
| Timestamp, | ||
|
|
@@ -167,31 +187,56 @@ SETTINGS index_granularity=8192, ttl_only_drop_parts = 1; | |
| Body, | ||
| ResourceAttributes, | ||
| LogAttributes | ||
| ) VALUES ( | ||
| ?, | ||
| ?, | ||
| ?, | ||
| ?, | ||
| ?, | ||
| ?, | ||
| ?, | ||
| ?, | ||
| ?, | ||
| ? | ||
| )` | ||
| )` | ||
| ) | ||
|
|
||
| var driverName = "clickhouse" // for testing | ||
|
|
||
| // newClickhouseClient create a clickhouse client. | ||
| func newClickhouseClient(cfg *Config) (*sql.DB, error) { | ||
| // newClickHouseClient create a clickhouse client. | ||
| // used by metrics and traces: | ||
| func newClickHouseClient(cfg *Config) (*sql.DB, error) { | ||
| db, err := cfg.buildDB(cfg.Database) | ||
| if err != nil { | ||
| return nil, err | ||
| } | ||
| return db, nil | ||
| } | ||
|
|
||
| // used by logs: | ||
| func newClickHouseConn(cfg *Config) (*sql.DB, error) { | ||
| endpoint := cfg.Endpoint | ||
|
|
||
| if len(cfg.ConnectionParams) > 0 { | ||
| values := make(url.Values, len(cfg.ConnectionParams)) | ||
| for k, v := range cfg.ConnectionParams { | ||
| values.Add(k, v) | ||
| } | ||
|
|
||
| if !strings.Contains(endpoint, "?") { | ||
| endpoint += "?" | ||
| } else if !strings.HasSuffix(endpoint, "&") { | ||
| endpoint += "&" | ||
| } | ||
|
|
||
| endpoint += values.Encode() | ||
| } | ||
|
|
||
| opts, err := clickhouse.ParseDSN(endpoint) | ||
| if err != nil { | ||
| return nil, fmt.Errorf("unable to parse endpoint: %w", err) | ||
| } | ||
|
|
||
| opts.Auth = clickhouse.Auth{ | ||
| Database: cfg.Database, | ||
| Username: cfg.Username, | ||
| Password: cfg.Password, | ||
| } | ||
|
|
||
| // can return a "bad" connection if misconfigured, we won't know | ||
| // until a Ping, Exec, etc.. is done | ||
| return clickhouse.OpenDB(opts), nil | ||
| } | ||
|
|
||
| func createDatabase(ctx context.Context, cfg *Config) error { | ||
| // use default database to create new database | ||
| if cfg.Database == defaultDatabase { | ||
|
|
@@ -231,17 +276,3 @@ func renderCreateLogsTableSQL(cfg *Config) string { | |
| func renderInsertLogsSQL(cfg *Config) string { | ||
| return fmt.Sprintf(insertLogsSQLTemplate, cfg.LogsTableName) | ||
| } | ||
|
|
||
| func doWithTx(_ context.Context, db *sql.DB, fn func(tx *sql.Tx) error) error { | ||
|
Collaborator
Author
There was a problem hiding this comment. Choose a reason for hiding this commentThe reason will be displayed to describe this comment to others. Learn more. this was not the proper way, they were using a tx but they were using an insert per row which is not correct, it's one prepared insert and then you append each row to the batch using |
||
| tx, err := db.Begin() | ||
| if err != nil { | ||
| return fmt.Errorf("db.Begin: %w", err) | ||
| } | ||
| defer func() { | ||
| _ = tx.Rollback() | ||
| }() | ||
| if err := fn(tx); err != nil { | ||
| return err | ||
| } | ||
| return tx.Commit() | ||
| } | ||
There was a problem hiding this comment.
Choose a reason for hiding this comment
The reason will be displayed to describe this comment to others. Learn more.
their Dockerfiles are confusing... the helm chart was expecting to run the command
/otelcol-contribbut the binary that gets built and copied into the image is/otelcontribcol?